Skip to content

Commit e080b9e

Browse files
committed
Reduce run time of StorageUtils.updateRddInfo to near constant
With the existing changes in the PR, the StorageListener still needed to iterate through all the blocks within a single RDD. Although this is already much better than before, it is still slow if a single RDD has many partitions. This commit further reduces the run time of StorageUtils.updateRddInfo to near constant. It achieves this by incrementally updating the storage information of each RDD (memory, disk, tachyon sizes) incrementally, rather than all at once when the caller demands it. A preliminary benchmark shows that the event queue length never exceeds 600 even for caching 10000 partitions within a single RDD. An important TODO is to add tests for the new code, as well as for the StorageListener, the source of the storage information on the UI.
1 parent 2c3ef6a commit e080b9e

File tree

4 files changed

+114
-94
lines changed

4 files changed

+114
-94
lines changed

core/src/main/scala/org/apache/spark/storage/BlockManagerMasterActor.scala

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -424,6 +424,11 @@ case class BlockStatus(
424424
def isCached: Boolean = memSize + diskSize + tachyonSize > 0
425425
}
426426

427+
@DeveloperApi
428+
object BlockStatus {
429+
def empty: BlockStatus = BlockStatus(StorageLevel.NONE, 0L, 0L, 0L)
430+
}
431+
427432
private[spark] class BlockManagerInfo(
428433
val blockManagerId: BlockManagerId,
429434
timeMs: Long,

core/src/main/scala/org/apache/spark/storage/StorageUtils.scala

Lines changed: 104 additions & 49 deletions
Original file line numberDiff line numberDiff line change
@@ -20,35 +20,45 @@ package org.apache.spark.storage
2020
import scala.collection.Map
2121
import scala.collection.mutable
2222

23+
import org.apache.spark.SparkException
2324
import org.apache.spark.annotation.DeveloperApi
2425

2526
/**
2627
* :: DeveloperApi ::
27-
* Storage information for each BlockManager. This class assumes BlockId and BlockStatus are
28-
* immutable, such that the consumers of this class will not mutate the source of the information.
28+
* Storage information for each BlockManager.
29+
*
30+
* This class assumes BlockId and BlockStatus are immutable, such that the consumers of this
31+
* class cannot mutate the source of the information. Accesses are not thread-safe.
2932
*/
3033
@DeveloperApi
3134
class StorageStatus(val blockManagerId: BlockManagerId, val maxMem: Long) {
3235

3336
/**
3437
* Internal representation of the blocks stored in this block manager.
3538
*
36-
* Common consumption patterns of these blocks include
37-
* (1) selecting all blocks,
38-
* (2) selecting only RDD blocks or,
39-
* (3) selecting only the blocks that belong to a specific RDD
40-
*
41-
* If we are only interested in a fraction of the blocks, as in (2) and (3), we should avoid
42-
* linearly scanning through all the blocks, which could be expensive if there are thousands
43-
* of blocks on each block manager. We achieve this by storing RDD blocks and non-RDD blocks
44-
* separately. In particular, RDD blocks are stored in a map indexed by RDD IDs, so we can
45-
* filter out the blocks of interest quickly.
46-
*
39+
* A common consumption pattern is to access only the blocks that belong to a specific RDD.
40+
* For this use case, we should avoid linearly scanning through all the blocks, which could
41+
* be expensive if there are thousands of blocks on each block manager. Thus, we need to store
42+
* RDD blocks and non-RDD blocks separately. In particular, we store RDD blocks in a map
43+
* indexed by RDD IDs, so we can filter out the blocks of interest quickly.
44+
4745
* These collections should only be mutated through the add/update/removeBlock methods.
4846
*/
4947
private val _rddBlocks = new mutable.HashMap[Int, mutable.Map[BlockId, BlockStatus]]
5048
private val _nonRddBlocks = new mutable.HashMap[BlockId, BlockStatus]
5149

50+
/**
51+
* A map of storage information associated with each RDD.
52+
*
53+
* The key is the ID of the RDD, and the value is a 4-tuple of the following:
54+
* (size in memory, size on disk, size in tachyon, storage level)
55+
*
56+
* This is updated incrementally on each block added, updated or removed, so as to avoid
57+
* linearly scanning through all the blocks within an RDD if we're only interested in a
58+
* given RDD's storage information.
59+
*/
60+
private val _rddStorageInfo = new mutable.HashMap[Int, (Long, Long, Long, StorageLevel)]
61+
5262
/**
5363
* Instantiate a StorageStatus with the given initial blocks. This essentially makes a copy of
5464
* the original blocks map such that the fate of this storage status is not tied to the source.
@@ -79,6 +89,14 @@ class StorageStatus(val blockManagerId: BlockManagerId, val maxMem: Long) {
7989
def addBlock(blockId: BlockId, blockStatus: BlockStatus): Unit = {
8090
blockId match {
8191
case RDDBlockId(rddId, _) =>
92+
// Update the storage info of the RDD, keeping track of any existing status for this block
93+
val oldBlockStatus = getBlock(blockId).getOrElse(BlockStatus.empty)
94+
val changeInMem = blockStatus.memSize - oldBlockStatus.memSize
95+
val changeInDisk = blockStatus.diskSize - oldBlockStatus.diskSize
96+
val changeInTachyon = blockStatus.tachyonSize - oldBlockStatus.tachyonSize
97+
val level = blockStatus.storageLevel
98+
updateRddStorageInfo(rddId, changeInMem, changeInDisk, changeInTachyon, level)
99+
// Actually add the block itself
82100
_rddBlocks.getOrElseUpdate(rddId, new mutable.HashMap)(blockId) = blockStatus
83101
case _ =>
84102
_nonRddBlocks(blockId) = blockStatus
@@ -94,6 +112,11 @@ class StorageStatus(val blockManagerId: BlockManagerId, val maxMem: Long) {
94112
def removeBlock(blockId: BlockId): Option[BlockStatus] = {
95113
blockId match {
96114
case RDDBlockId(rddId, _) =>
115+
// Update the storage info of the RDD if the block to remove exists
116+
getBlock(blockId).foreach { s =>
117+
updateRddStorageInfo(rddId, -s.memSize, -s.diskSize, -s.tachyonSize, StorageLevel.NONE)
118+
}
119+
// Actually remove the block, if it exists
97120
if (_rddBlocks.contains(rddId)) {
98121
val removed = _rddBlocks(rddId).remove(blockId)
99122
// If the given RDD has no more blocks left, remove the RDD
@@ -136,33 +159,79 @@ class StorageStatus(val blockManagerId: BlockManagerId, val maxMem: Long) {
136159
}
137160

138161
/**
139-
* Return the number of blocks stored in this block manager in O(rdds) time.
162+
* Return the number of blocks stored in this block manager in O(RDDs) time.
140163
* Note that this is much faster than `this.blocks.size`, which is O(blocks) time.
141164
*/
142165
def numBlocks: Int = {
143166
_nonRddBlocks.size + _rddBlocks.values.map(_.size).reduceOption(_ + _).getOrElse(0)
144167
}
145168

169+
/**
170+
* Return the number of RDD blocks stored in this block manager in O(RDDs) time.
171+
* Note that this is much faster than `this.rddBlocks.size`, which is O(RDD blocks) time.
172+
*/
173+
def numRddBlocks: Int = _rddBlocks.keys.map(numRddBlocksById).reduceOption(_ + _).getOrElse(0)
174+
175+
/**
176+
* Return the number of blocks that belong to the given RDD in O(1) time.
177+
* Note that this is much faster than `this.rddBlocksById(rddId).size`, which is
178+
* O(blocks in this RDD) time.
179+
*/
180+
def numRddBlocksById(rddId: Int): Int = _rddBlocks.get(rddId).map(_.size).getOrElse(0)
181+
146182
/** Return the memory used by this block manager. */
147-
def memUsed: Long = memUsed(blocks)
183+
def memUsed: Long = blocks.values.map(_.memSize).reduceOption(_ + _).getOrElse(0L)
148184

149185
/** Return the memory used by the given RDD in this block manager. */
150-
def memUsedByRDD(rddId: Int): Long = memUsed(rddBlocksById(rddId))
186+
def memUsedByRDD(rddId: Int): Long = _rddStorageInfo.get(rddId).map(_._1).getOrElse(0L)
151187

152188
/** Return the memory remaining in this block manager. */
153189
def memRemaining: Long = maxMem - memUsed
154190

155191
/** Return the disk space used by this block manager. */
156-
def diskUsed: Long = diskUsed(blocks)
192+
def diskUsed: Long = blocks.values.map(_.diskSize).reduceOption(_ + _).getOrElse(0L)
157193

158194
/** Return the disk space used by the given RDD in this block manager. */
159-
def diskUsedByRDD(rddId: Int): Long = diskUsed(rddBlocksById(rddId))
195+
def diskUsedByRDD(rddId: Int): Long = _rddStorageInfo.get(rddId).map(_._2).getOrElse(0L)
196+
197+
/** Return the off-heap space used by this block manager. */
198+
def offHeapUsed: Long = blocks.values.map(_.tachyonSize).reduceOption(_ + _).getOrElse(0L)
199+
200+
/** Return the off-heap space used by the given RDD in this block manager. */
201+
def offHeapUsedByRdd(rddId: Int): Long = _rddStorageInfo.get(rddId).map(_._3).getOrElse(0L)
202+
203+
/** Return the storage level, if any, used by the given RDD in this block manager. */
204+
def rddStorageLevel(rddId: Int): Option[StorageLevel] = _rddStorageInfo.get(rddId).map(_._4)
160205

161-
// Helper methods for computing memory and disk usages
162-
private def memUsed(_blocks: Map[BlockId, BlockStatus]): Long =
163-
_blocks.values.map(_.memSize).reduceOption(_ + _).getOrElse(0L)
164-
private def diskUsed(_blocks: Map[BlockId, BlockStatus]): Long =
165-
_blocks.values.map(_.diskSize).reduceOption(_ + _).getOrElse(0L)
206+
/**
207+
* Helper function to update the given RDD's storage information based on the
208+
* (possibly negative) changes in memory, disk, and off-heap memory usages.
209+
*/
210+
private def updateRddStorageInfo(
211+
rddId: Int,
212+
changeInMem: Long,
213+
changeInDisk: Long,
214+
changeInTachyon: Long,
215+
storageLevel: StorageLevel): Unit = {
216+
val emptyRddInfo = (0L, 0L, 0L, StorageLevel.NONE)
217+
val oldRddInfo = _rddStorageInfo.getOrElse(rddId, emptyRddInfo)
218+
val newRddInfo = oldRddInfo match {
219+
case (oldRddMem, oldRddDisk, oldRddTachyon, _) =>
220+
val newRddMem = math.max(oldRddMem + changeInMem, 0L)
221+
val newRddDisk = math.max(oldRddDisk + changeInDisk, 0L)
222+
val newRddTachyon = math.max(oldRddTachyon + changeInTachyon, 0L)
223+
(newRddMem, newRddDisk, newRddTachyon, storageLevel)
224+
case _ =>
225+
// Should never happen
226+
throw new SparkException(s"Existing information for $rddId is not of expected type")
227+
}
228+
// If this RDD is no longer persisted, remove it
229+
if (newRddInfo._1 + newRddInfo._2 + newRddInfo._3 == 0) {
230+
_rddStorageInfo.remove(rddId)
231+
} else {
232+
_rddStorageInfo(rddId) = newRddInfo
233+
}
234+
}
166235
}
167236

168237
/** Helper methods for storage-related objects. */
@@ -172,32 +241,20 @@ private[spark] object StorageUtils {
172241
* Update the given list of RDDInfo with the given list of storage statuses.
173242
* This method overwrites the old values stored in the RDDInfo's.
174243
*/
175-
def updateRddInfo(
176-
rddInfos: Seq[RDDInfo],
177-
storageStatuses: Seq[StorageStatus],
178-
updatedBlocks: Seq[(BlockId, BlockStatus)] = Seq.empty): Unit = {
179-
244+
def updateRddInfo(rddInfos: Seq[RDDInfo], statuses: Seq[StorageStatus]): Unit = {
180245
rddInfos.foreach { rddInfo =>
181246
val rddId = rddInfo.id
182-
183-
// Collect all block statuses that belong to the given RDD
184-
val newBlocks = updatedBlocks.filter { case (bid, _) =>
185-
bid.asRDDId.filter(_.rddId == rddId).isDefined
186-
}
187-
val newBlockIds = newBlocks.map { case (bid, _) => bid }.toSet
188-
val oldBlocks = storageStatuses
189-
.flatMap(_.rddBlocksById(rddId))
190-
.filter { case (bid, _) => !newBlockIds.contains(bid) } // avoid double counting
191-
val blocks = (oldBlocks ++ newBlocks).map { case (_, bstatus) => bstatus }
192-
val persistedBlocks = blocks.filter(_.isCached)
193-
194247
// Assume all blocks belonging to the same RDD have the same storage level
195-
val storageLevel = blocks.headOption.map(_.storageLevel).getOrElse(StorageLevel.NONE)
196-
val memSize = persistedBlocks.map(_.memSize).reduceOption(_ + _).getOrElse(0L)
197-
val diskSize = persistedBlocks.map(_.diskSize).reduceOption(_ + _).getOrElse(0L)
198-
val tachyonSize = persistedBlocks.map(_.tachyonSize).reduceOption(_ + _).getOrElse(0L)
248+
val storageLevel = statuses
249+
.map(_.rddStorageLevel(rddId)).flatMap(s => s).headOption.getOrElse(StorageLevel.NONE)
250+
val numCachedPartitions = statuses
251+
.map(_.numRddBlocksById(rddId)).reduceOption(_ + _).getOrElse(0)
252+
val memSize = statuses.map(_.memUsedByRDD(rddId)).reduceOption(_ + _).getOrElse(0L)
253+
val diskSize = statuses.map(_.diskUsedByRDD(rddId)).reduceOption(_ + _).getOrElse(0L)
254+
val tachyonSize = statuses.map(_.offHeapUsedByRdd(rddId)).reduceOption(_ + _).getOrElse(0L)
255+
199256
rddInfo.storageLevel = storageLevel
200-
rddInfo.numCachedPartitions = persistedBlocks.length
257+
rddInfo.numCachedPartitions = numCachedPartitions
201258
rddInfo.memSize = memSize
202259
rddInfo.diskSize = diskSize
203260
rddInfo.tachyonSize = tachyonSize
@@ -207,11 +264,9 @@ private[spark] object StorageUtils {
207264
/**
208265
* Return mapping from block ID to its locations for each block that belongs to the given RDD.
209266
*/
210-
def getRddBlockLocations(
211-
storageStatuses: Seq[StorageStatus],
212-
rddId: Int): Map[BlockId, Seq[String]] = {
267+
def getRddBlockLocations(statuses: Seq[StorageStatus], rddId: Int): Map[BlockId, Seq[String]] = {
213268
val blockLocations = new mutable.HashMap[BlockId, mutable.ListBuffer[String]]
214-
storageStatuses.foreach { status =>
269+
statuses.foreach { status =>
215270
status.rddBlocksById(rddId).foreach { case (bid, _) =>
216271
val location = status.blockManagerId.hostPort
217272
blockLocations.getOrElseUpdate(bid, mutable.ListBuffer.empty) += location

core/src/main/scala/org/apache/spark/ui/storage/StorageTab.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -52,7 +52,7 @@ class StorageListener(storageStatusListener: StorageStatusListener) extends Spar
5252
private def updateRDDInfo(updatedBlocks: Seq[(BlockId, BlockStatus)]): Unit = {
5353
val rddIdsToUpdate = updatedBlocks.flatMap { case (bid, _) => bid.asRDDId.map(_.rddId) }.toSet
5454
val rddInfosToUpdate = _rddInfoMap.values.toSeq.filter { s => rddIdsToUpdate.contains(s.id) }
55-
StorageUtils.updateRddInfo(rddInfosToUpdate, storageStatusList, updatedBlocks)
55+
StorageUtils.updateRddInfo(rddInfosToUpdate, storageStatusList)
5656
}
5757

5858
/**

core/src/test/scala/org/apache/spark/storage/StorageSuite.scala

Lines changed: 4 additions & 44 deletions
Original file line numberDiff line numberDiff line change
@@ -228,56 +228,16 @@ class StorageSuite extends FunSuite {
228228
val storageStatuses = stockStorageStatuses
229229
val rddInfos = stockRDDInfos
230230
StorageUtils.updateRddInfo(rddInfos, storageStatuses)
231+
assert(rddInfos(0).storageLevel === memAndDisk)
231232
assert(rddInfos(0).numCachedPartitions === 5)
232233
assert(rddInfos(0).memSize === 5L)
233234
assert(rddInfos(0).diskSize === 10L)
235+
assert(rddInfos(0).tachyonSize === 0L)
236+
assert(rddInfos(1).storageLevel === memAndDisk)
234237
assert(rddInfos(1).numCachedPartitions === 3)
235238
assert(rddInfos(1).memSize === 3L)
236239
assert(rddInfos(1).diskSize === 6L)
237-
}
238-
239-
test("StorageUtils.updateRddInfo with updated blocks") {
240-
val storageStatuses = stockStorageStatuses
241-
val rddInfos = stockRDDInfos
242-
243-
// Drop 3 blocks from RDD 0, and cache more of RDD 1
244-
val updatedBlocks1 = Seq(
245-
(RDDBlockId(0, 0), BlockStatus(memAndDisk, 0L, 0L, 0L)),
246-
(RDDBlockId(0, 1), BlockStatus(memAndDisk, 0L, 0L, 0L)),
247-
(RDDBlockId(0, 2), BlockStatus(memAndDisk, 0L, 0L, 0L)),
248-
(RDDBlockId(1, 0), BlockStatus(memAndDisk, 100L, 100L, 0L)),
249-
(RDDBlockId(1, 100), BlockStatus(memAndDisk, 100L, 100L, 0L))
250-
)
251-
StorageUtils.updateRddInfo(rddInfos, storageStatuses, updatedBlocks1)
252-
assert(rddInfos(0).numCachedPartitions === 2)
253-
assert(rddInfos(0).memSize === 2L)
254-
assert(rddInfos(0).diskSize === 4L)
255-
assert(rddInfos(1).numCachedPartitions === 4)
256-
assert(rddInfos(1).memSize === 202L)
257-
assert(rddInfos(1).diskSize === 204L)
258-
259-
// Actually update storage statuses so we can chain the calls to StorageUtils.updateRddInfo
260-
updatedBlocks1.foreach { case (bid, bstatus) =>
261-
storageStatuses.find(_.containsBlock(bid)) match {
262-
case Some(s) => s.updateBlock(bid, bstatus)
263-
case None => storageStatuses(0).addBlock(bid, bstatus) // arbitrarily pick the first
264-
}
265-
}
266-
267-
// Drop all of RDD 1, following previous updates
268-
val updatedBlocks2 = Seq(
269-
(RDDBlockId(1, 0), BlockStatus(memAndDisk, 0L, 0L, 0L)),
270-
(RDDBlockId(1, 1), BlockStatus(memAndDisk, 0L, 0L, 0L)),
271-
(RDDBlockId(1, 2), BlockStatus(memAndDisk, 0L, 0L, 0L)),
272-
(RDDBlockId(1, 100), BlockStatus(memAndDisk, 0L, 0L, 0L))
273-
)
274-
StorageUtils.updateRddInfo(rddInfos, storageStatuses, updatedBlocks2)
275-
assert(rddInfos(0).numCachedPartitions === 2)
276-
assert(rddInfos(0).memSize === 2L)
277-
assert(rddInfos(0).diskSize === 4L)
278-
assert(rddInfos(1).numCachedPartitions === 0)
279-
assert(rddInfos(1).memSize === 0L)
280-
assert(rddInfos(1).diskSize === 0L)
240+
assert(rddInfos(1).tachyonSize === 0L)
281241
}
282242

283243
test("StorageUtils.getRddBlockLocations") {

0 commit comments

Comments
 (0)