@@ -36,42 +36,52 @@ import org.apache.spark.unsafe.memory.MemoryAllocator
3636private [spark] abstract class MemoryManager (
3737 conf : SparkConf ,
3838 numCores : Int ,
39- storageMemory : Long ,
39+ onHeapStorageMemory : Long ,
4040 onHeapExecutionMemory : Long ) extends Logging {
4141
4242 // -- Methods related to memory allocation policies and bookkeeping ------------------------------
4343
4444 @ GuardedBy (" this" )
45- protected val storageMemoryPool = new StorageMemoryPool (this )
45+ protected val onHeapStorageMemoryPool = new StorageMemoryPool (this , MemoryMode . ON_HEAP )
4646 @ GuardedBy (" this" )
47- protected val onHeapExecutionMemoryPool = new ExecutionMemoryPool (this , " on-heap execution " )
47+ protected val offHeapStorageMemoryPool = new StorageMemoryPool (this , MemoryMode . OFF_HEAP )
4848 @ GuardedBy (" this" )
49- protected val offHeapExecutionMemoryPool = new ExecutionMemoryPool (this , " off-heap execution" )
49+ protected val onHeapExecutionMemoryPool = new ExecutionMemoryPool (this , MemoryMode .ON_HEAP )
50+ @ GuardedBy (" this" )
51+ protected val offHeapExecutionMemoryPool = new ExecutionMemoryPool (this , MemoryMode .OFF_HEAP )
5052
51- storageMemoryPool .incrementPoolSize(storageMemory )
53+ onHeapStorageMemoryPool .incrementPoolSize(onHeapStorageMemory )
5254 onHeapExecutionMemoryPool.incrementPoolSize(onHeapExecutionMemory)
53- offHeapExecutionMemoryPool.incrementPoolSize(conf.getSizeAsBytes(" spark.memory.offHeap.size" , 0 ))
55+
56+ protected [this ] val maxOffHeapMemory = conf.getSizeAsBytes(" spark.memory.offHeap.size" , 0 )
57+ protected [this ] val offHeapStorageMemory =
58+ (maxOffHeapMemory * conf.getDouble(" spark.memory.storageFraction" , 0.5 )).toLong
59+
60+ offHeapExecutionMemoryPool.incrementPoolSize(maxOffHeapMemory - offHeapStorageMemory)
61+ offHeapStorageMemoryPool.incrementPoolSize(offHeapStorageMemory)
5462
5563 /**
5664 * Total available memory for storage, in bytes. This amount can vary over time, depending on
5765 * the MemoryManager implementation.
5866 * In this model, this is equivalent to the amount of memory not occupied by execution.
5967 */
60- def maxStorageMemory : Long
68+ def maxOnHeapStorageMemory : Long
6169
6270 /**
6371 * Set the [[MemoryStore ]] used by this manager to evict cached blocks.
6472 * This must be set after construction due to initialization ordering constraints.
6573 */
6674 final def setMemoryStore (store : MemoryStore ): Unit = synchronized {
67- storageMemoryPool.setMemoryStore(store)
75+ onHeapStorageMemoryPool.setMemoryStore(store)
76+ offHeapStorageMemoryPool.setMemoryStore(store)
6877 }
6978
7079 /**
7180 * Acquire N bytes of memory to cache the given block, evicting existing ones if necessary.
81+ *
7282 * @return whether all N bytes were successfully granted.
7383 */
74- def acquireStorageMemory (blockId : BlockId , numBytes : Long ): Boolean
84+ def acquireStorageMemory (blockId : BlockId , numBytes : Long , memoryMode : MemoryMode ): Boolean
7585
7686 /**
7787 * Acquire N bytes of memory to unroll the given block, evicting existing ones if necessary.
@@ -82,7 +92,7 @@ private[spark] abstract class MemoryManager(
8292 *
8393 * @return whether all N bytes were successfully granted.
8494 */
85- def acquireUnrollMemory (blockId : BlockId , numBytes : Long ): Boolean
95+ def acquireUnrollMemory (blockId : BlockId , numBytes : Long , memoryMode : MemoryMode ): Boolean
8696
8797 /**
8898 * Try to acquire up to `numBytes` of execution memory for the current task and return the
@@ -126,22 +136,26 @@ private[spark] abstract class MemoryManager(
126136 /**
127137 * Release N bytes of storage memory.
128138 */
129- def releaseStorageMemory (numBytes : Long ): Unit = synchronized {
130- storageMemoryPool.releaseMemory(numBytes)
139+ def releaseStorageMemory (numBytes : Long , memoryMode : MemoryMode ): Unit = synchronized {
140+ memoryMode match {
141+ case MemoryMode .ON_HEAP => onHeapStorageMemoryPool.releaseMemory(numBytes)
142+ case MemoryMode .OFF_HEAP => offHeapStorageMemoryPool.releaseMemory(numBytes)
143+ }
131144 }
132145
133146 /**
134147 * Release all storage memory acquired.
135148 */
136149 final def releaseAllStorageMemory (): Unit = synchronized {
137- storageMemoryPool.releaseAllMemory()
150+ onHeapStorageMemoryPool.releaseAllMemory()
151+ offHeapStorageMemoryPool.releaseAllMemory()
138152 }
139153
140154 /**
141155 * Release N bytes of unroll memory.
142156 */
143- final def releaseUnrollMemory (numBytes : Long ): Unit = synchronized {
144- releaseStorageMemory(numBytes)
157+ final def releaseUnrollMemory (numBytes : Long , memoryMode : MemoryMode ): Unit = synchronized {
158+ releaseStorageMemory(numBytes, memoryMode )
145159 }
146160
147161 /**
@@ -155,7 +169,7 @@ private[spark] abstract class MemoryManager(
155169 * Storage memory currently in use, in bytes.
156170 */
157171 final def storageMemoryUsed : Long = synchronized {
158- storageMemoryPool .memoryUsed
172+ onHeapStorageMemoryPool.memoryUsed + offHeapStorageMemoryPool .memoryUsed
159173 }
160174
161175 /**
0 commit comments