Skip to content

Commit d0ada7b

Browse files
author
Davies Liu
committed
force spilling
1 parent 616be29 commit d0ada7b

File tree

23 files changed

+861
-640
lines changed

23 files changed

+861
-640
lines changed
Lines changed: 100 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,100 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.memory;
19+
20+
21+
import java.io.IOException;
22+
23+
import org.apache.spark.unsafe.memory.MemoryBlock;
24+
25+
26+
/**
27+
* An memory consumer of TaskMemoryManager, which support spilling.
28+
*/
29+
public class MemoryConsumer {
30+
31+
private TaskMemoryManager memoryManager;
32+
private long pageSize;
33+
34+
protected MemoryConsumer(TaskMemoryManager memoryManager, long pageSize) {
35+
this.memoryManager = memoryManager;
36+
this.pageSize = pageSize;
37+
}
38+
39+
protected MemoryConsumer(TaskMemoryManager memoryManager) {
40+
this(memoryManager, memoryManager.pageSizeBytes());
41+
}
42+
43+
/**
44+
* Spill some data to disk to release memory, which will be called by TaskMemoryManager
45+
* when there is not enough memory for the task.
46+
*
47+
* @param size the amount of memory should be released
48+
* @return the amount of released memory in bytes
49+
* @throws IOException
50+
*/
51+
public long spill(long size) throws IOException {
52+
return 0L;
53+
}
54+
55+
/**
56+
* Acquire `size` bytes memory.
57+
*
58+
* If there is not enough memory, throws IOException.
59+
*
60+
* @throws IOException
61+
*/
62+
protected void acquireMemory(long size) throws IOException {
63+
long got = memoryManager.acquireExecutionMemory(size, this);
64+
if (got < size) {
65+
throw new IOException("Could not acquire " + size + " bytes of memory " + got);
66+
}
67+
}
68+
69+
/**
70+
* Release amount of memory.
71+
*/
72+
protected void releaseMemory(long size) {
73+
memoryManager.releaseExecutionMemory(size, this);
74+
}
75+
76+
/**
77+
* Allocate a memory block with at least `required` bytes.
78+
*
79+
* Throws IOException if there is not enough memory.
80+
*
81+
* @throws IOException
82+
*/
83+
protected MemoryBlock allocatePage(long required) throws IOException {
84+
MemoryBlock page = memoryManager.allocatePage(Math.max(pageSize, required), this);
85+
if (page == null || page.size() < required) {
86+
if (page != null) {
87+
freePage(page);
88+
}
89+
throw new IOException("Unable to acquire " + required + " bytes of memory");
90+
}
91+
return page;
92+
}
93+
94+
/**
95+
* Free a memory block.
96+
*/
97+
protected void freePage(MemoryBlock page) {
98+
memoryManager.freePage(page, this);
99+
}
100+
}

core/src/main/java/org/apache/spark/memory/TaskMemoryManager.java

Lines changed: 95 additions & 11 deletions
Original file line numberDiff line numberDiff line change
@@ -17,7 +17,9 @@
1717

1818
package org.apache.spark.memory;
1919

20-
import java.util.*;
20+
import java.io.IOException;
21+
import java.util.BitSet;
22+
import java.util.HashMap;
2123

2224
import com.google.common.annotations.VisibleForTesting;
2325
import org.slf4j.Logger;
@@ -100,13 +102,19 @@ public class TaskMemoryManager {
100102
*/
101103
private final boolean inHeap;
102104

105+
/**
106+
* The size of memory granted to each consumer.
107+
*/
108+
private HashMap<MemoryConsumer, Long> consumers;
109+
103110
/**
104111
* Construct a new TaskMemoryManager.
105112
*/
106113
public TaskMemoryManager(MemoryManager memoryManager, long taskAttemptId) {
107114
this.inHeap = memoryManager.tungstenMemoryIsAllocatedInHeap();
108115
this.memoryManager = memoryManager;
109116
this.taskAttemptId = taskAttemptId;
117+
this.consumers = new HashMap<>();
110118
}
111119

112120
/**
@@ -117,13 +125,75 @@ public long acquireExecutionMemory(long size) {
117125
return memoryManager.acquireExecutionMemory(size, taskAttemptId);
118126
}
119127

128+
/**
129+
* Acquire N bytes of memory for a consumer. If there is no enough memory, it will call
130+
* spill() of consumers to release more memory.
131+
*
132+
* @return number of bytes successfully granted (<= N).
133+
*/
134+
public long acquireExecutionMemory(long size, MemoryConsumer consumer) throws IOException {
135+
synchronized (this) {
136+
long got = acquireExecutionMemory(size);
137+
138+
if (got < size && consumer != null) {
139+
// call spill() on itself to release some memory
140+
consumer.spill(size - got);
141+
got += acquireExecutionMemory(size - got);
142+
143+
if (got < size) {
144+
long needed = size - got;
145+
// call spill() on other consumers to release memory
146+
for (MemoryConsumer c: consumers.keySet()) {
147+
if (c != consumer) {
148+
needed -= c.spill(size - got);
149+
if (needed < 0) {
150+
break;
151+
}
152+
}
153+
}
154+
got += acquireExecutionMemory(size - got);
155+
}
156+
}
157+
158+
long old = 0L;
159+
if (consumers.containsKey(consumer)) {
160+
old = consumers.get(consumer);
161+
}
162+
consumers.put(consumer, got + old);
163+
164+
return got;
165+
}
166+
}
167+
120168
/**
121169
* Release N bytes of execution memory.
122170
*/
123171
public void releaseExecutionMemory(long size) {
124172
memoryManager.releaseExecutionMemory(size, taskAttemptId);
125173
}
126174

175+
/**
176+
* Release N bytes of execution memory for a MemoryConsumer.
177+
*/
178+
public void releaseExecutionMemory(long size, MemoryConsumer consumer) {
179+
synchronized (this) {
180+
if (consumer != null && consumers.containsKey(consumer)) {
181+
long old = consumers.get(consumer);
182+
if (old > size) {
183+
consumers.put(consumer, old - size);
184+
} else {
185+
if (old < size) {
186+
// TODO
187+
}
188+
consumers.remove(consumer);
189+
}
190+
} else {
191+
// TODO
192+
}
193+
memoryManager.releaseExecutionMemory(size, taskAttemptId);
194+
}
195+
}
196+
127197
public long pageSizeBytes() {
128198
return memoryManager.pageSizeBytes();
129199
}
@@ -134,12 +204,27 @@ public long pageSizeBytes() {
134204
*
135205
* Returns `null` if there was not enough memory to allocate the page.
136206
*/
137-
public MemoryBlock allocatePage(long size) {
207+
public MemoryBlock allocatePage(long size) throws IOException {
208+
return allocatePage(size, null);
209+
}
210+
211+
/**
212+
* Allocate a block of memory that will be tracked in the MemoryManager's page table; this is
213+
* intended for allocating large blocks of Tungsten memory that will be shared between operators.
214+
*
215+
* Returns `null` if there was not enough memory to allocate the page.
216+
*/
217+
public MemoryBlock allocatePage(long size, MemoryConsumer consumer) throws IOException {
138218
if (size > MAXIMUM_PAGE_SIZE_BYTES) {
139219
throw new IllegalArgumentException(
140220
"Cannot allocate a page with more than " + MAXIMUM_PAGE_SIZE_BYTES + " bytes");
141221
}
142222

223+
long acquired = acquireExecutionMemory(size, consumer);
224+
if (acquired <= 0) {
225+
return null;
226+
}
227+
143228
final int pageNumber;
144229
synchronized (this) {
145230
pageNumber = allocatedPages.nextClearBit(0);
@@ -149,14 +234,6 @@ public MemoryBlock allocatePage(long size) {
149234
}
150235
allocatedPages.set(pageNumber);
151236
}
152-
final long acquiredExecutionMemory = acquireExecutionMemory(size);
153-
if (acquiredExecutionMemory != size) {
154-
releaseExecutionMemory(acquiredExecutionMemory);
155-
synchronized (this) {
156-
allocatedPages.clear(pageNumber);
157-
}
158-
return null;
159-
}
160237
final MemoryBlock page = memoryManager.tungstenMemoryAllocator().allocate(size);
161238
page.pageNumber = pageNumber;
162239
pageTable[pageNumber] = page;
@@ -170,6 +247,13 @@ public MemoryBlock allocatePage(long size) {
170247
* Free a block of memory allocated via {@link TaskMemoryManager#allocatePage(long)}.
171248
*/
172249
public void freePage(MemoryBlock page) {
250+
freePage(page, null);
251+
}
252+
253+
/**
254+
* Free a block of memory allocated via {@link TaskMemoryManager#allocatePage(long)}.
255+
*/
256+
public void freePage(MemoryBlock page, MemoryConsumer consumer) {
173257
assert (page.pageNumber != -1) :
174258
"Called freePage() on memory that wasn't allocated with allocatePage()";
175259
assert(allocatedPages.get(page.pageNumber));
@@ -182,7 +266,7 @@ public void freePage(MemoryBlock page) {
182266
}
183267
long pageSize = page.size();
184268
memoryManager.tungstenMemoryAllocator().free(page);
185-
releaseExecutionMemory(pageSize);
269+
releaseExecutionMemory(pageSize, consumer);
186270
}
187271

188272
/**

0 commit comments

Comments
 (0)