Skip to content
This repository was archived by the owner on Jan 9, 2020. It is now read-only.

Commit 772e464

Browse files
Marcelo Vanzinsquito
authored andcommitted
[SPARK-20653][CORE] Add cleaning of old elements from the status store.
This change restores the functionality that keeps a limited number of different types (jobs, stages, etc) depending on configuration, to avoid the store growing indefinitely over time. The feature is implemented by creating a new type (ElementTrackingStore) that wraps a KVStore and allows triggers to be set up for when elements of a certain type meet a certain threshold. Triggers don't need to necessarily only delete elements, but the current API is set up in a way that makes that use case easier. The new store also has a trigger for the "close" call, which makes it easier for listeners to register code for cleaning things up and flushing partial state to the store. The old configurations for cleaning up the stored elements from the core and SQL UIs are now active again, and the old unit tests are re-enabled. Author: Marcelo Vanzin <[email protected]> Closes apache#19751 from vanzin/SPARK-20653.
1 parent fb3636b commit 772e464

File tree

22 files changed

+713
-81
lines changed

22 files changed

+713
-81
lines changed

core/src/main/scala/org/apache/spark/deploy/history/FsHistoryProvider.scala

Lines changed: 10 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -44,6 +44,7 @@ import org.apache.spark.scheduler.ReplayListenerBus._
4444
import org.apache.spark.status._
4545
import org.apache.spark.status.KVUtils._
4646
import org.apache.spark.status.api.v1.{ApplicationAttemptInfo, ApplicationInfo}
47+
import org.apache.spark.status.config._
4748
import org.apache.spark.ui.SparkUI
4849
import org.apache.spark.util.{Clock, SystemClock, ThreadUtils, Utils}
4950
import org.apache.spark.util.kvstore._
@@ -304,6 +305,9 @@ private[history] class FsHistoryProvider(conf: SparkConf, clock: Clock)
304305
val (kvstore, needReplay) = uiStorePath match {
305306
case Some(path) =>
306307
try {
308+
// The store path is not guaranteed to exist - maybe it hasn't been created, or was
309+
// invalidated because changes to the event log were detected. Need to replay in that
310+
// case.
307311
val _replay = !path.isDirectory()
308312
(createDiskStore(path, conf), _replay)
309313
} catch {
@@ -318,24 +322,23 @@ private[history] class FsHistoryProvider(conf: SparkConf, clock: Clock)
318322
(new InMemoryStore(), true)
319323
}
320324

325+
val trackingStore = new ElementTrackingStore(kvstore, conf)
321326
if (needReplay) {
322327
val replayBus = new ReplayListenerBus()
323-
val listener = new AppStatusListener(kvstore, conf, false,
328+
val listener = new AppStatusListener(trackingStore, conf, false,
324329
lastUpdateTime = Some(attempt.info.lastUpdated.getTime()))
325330
replayBus.addListener(listener)
326331
AppStatusPlugin.loadPlugins().foreach { plugin =>
327-
plugin.setupListeners(conf, kvstore, l => replayBus.addListener(l), false)
332+
plugin.setupListeners(conf, trackingStore, l => replayBus.addListener(l), false)
328333
}
329334
try {
330335
val fileStatus = fs.getFileStatus(new Path(logDir, attempt.logPath))
331336
replay(fileStatus, isApplicationCompleted(fileStatus), replayBus)
332-
listener.flush()
337+
trackingStore.close(false)
333338
} catch {
334339
case e: Exception =>
335-
try {
336-
kvstore.close()
337-
} catch {
338-
case _e: Exception => logInfo("Error closing store.", _e)
340+
Utils.tryLogNonFatalError {
341+
trackingStore.close()
339342
}
340343
uiStorePath.foreach(Utils.deleteRecursively)
341344
if (e.isInstanceOf[FileNotFoundException]) {

core/src/main/scala/org/apache/spark/internal/config/package.scala

Lines changed: 0 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -240,11 +240,6 @@ package object config {
240240
.stringConf
241241
.createOptional
242242

243-
// To limit memory usage, we only track information for a fixed number of tasks
244-
private[spark] val UI_RETAINED_TASKS = ConfigBuilder("spark.ui.retainedTasks")
245-
.intConf
246-
.createWithDefault(100000)
247-
248243
// To limit how many applications are shown in the History Server summary ui
249244
private[spark] val HISTORY_UI_MAX_APPS =
250245
ConfigBuilder("spark.history.ui.maxApplications").intConf.createWithDefault(Integer.MAX_VALUE)

core/src/main/scala/org/apache/spark/status/AppStatusListener.scala

Lines changed: 176 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -32,7 +32,6 @@ import org.apache.spark.status.api.v1
3232
import org.apache.spark.storage._
3333
import org.apache.spark.ui.SparkUI
3434
import org.apache.spark.ui.scope._
35-
import org.apache.spark.util.kvstore.KVStore
3635

3736
/**
3837
* A Spark listener that writes application information to a data store. The types written to the
@@ -42,7 +41,7 @@ import org.apache.spark.util.kvstore.KVStore
4241
* unfinished tasks can be more accurately calculated (see SPARK-21922).
4342
*/
4443
private[spark] class AppStatusListener(
45-
kvstore: KVStore,
44+
kvstore: ElementTrackingStore,
4645
conf: SparkConf,
4746
live: Boolean,
4847
lastUpdateTime: Option[Long] = None) extends SparkListener with Logging {
@@ -51,13 +50,15 @@ private[spark] class AppStatusListener(
5150

5251
private var sparkVersion = SPARK_VERSION
5352
private var appInfo: v1.ApplicationInfo = null
53+
private var appSummary = new AppSummary(0, 0)
5454
private var coresPerTask: Int = 1
5555

5656
// How often to update live entities. -1 means "never update" when replaying applications,
5757
// meaning only the last write will happen. For live applications, this avoids a few
5858
// operations that we can live without when rapidly processing incoming task events.
5959
private val liveUpdatePeriodNs = if (live) conf.get(LIVE_ENTITY_UPDATE_PERIOD) else -1L
6060

61+
private val maxTasksPerStage = conf.get(MAX_RETAINED_TASKS_PER_STAGE)
6162
private val maxGraphRootNodes = conf.get(MAX_RETAINED_ROOT_NODES)
6263

6364
// Keep track of live entities, so that task metrics can be efficiently updated (without
@@ -68,10 +69,25 @@ private[spark] class AppStatusListener(
6869
private val liveTasks = new HashMap[Long, LiveTask]()
6970
private val liveRDDs = new HashMap[Int, LiveRDD]()
7071
private val pools = new HashMap[String, SchedulerPool]()
72+
// Keep the active executor count as a separate variable to avoid having to do synchronization
73+
// around liveExecutors.
74+
@volatile private var activeExecutorCount = 0
7175

72-
override def onOtherEvent(event: SparkListenerEvent): Unit = event match {
73-
case SparkListenerLogStart(version) => sparkVersion = version
74-
case _ =>
76+
kvstore.addTrigger(classOf[ExecutorSummaryWrapper], conf.get(MAX_RETAINED_DEAD_EXECUTORS))
77+
{ count => cleanupExecutors(count) }
78+
79+
kvstore.addTrigger(classOf[JobDataWrapper], conf.get(MAX_RETAINED_JOBS)) { count =>
80+
cleanupJobs(count)
81+
}
82+
83+
kvstore.addTrigger(classOf[StageDataWrapper], conf.get(MAX_RETAINED_STAGES)) { count =>
84+
cleanupStages(count)
85+
}
86+
87+
kvstore.onFlush {
88+
if (!live) {
89+
flush()
90+
}
7591
}
7692

7793
override def onApplicationStart(event: SparkListenerApplicationStart): Unit = {
@@ -97,6 +113,7 @@ private[spark] class AppStatusListener(
97113
Seq(attempt))
98114

99115
kvstore.write(new ApplicationInfoWrapper(appInfo))
116+
kvstore.write(appSummary)
100117
}
101118

102119
override def onEnvironmentUpdate(event: SparkListenerEnvironmentUpdate): Unit = {
@@ -158,10 +175,11 @@ private[spark] class AppStatusListener(
158175
override def onExecutorRemoved(event: SparkListenerExecutorRemoved): Unit = {
159176
liveExecutors.remove(event.executorId).foreach { exec =>
160177
val now = System.nanoTime()
178+
activeExecutorCount = math.max(0, activeExecutorCount - 1)
161179
exec.isActive = false
162180
exec.removeTime = new Date(event.time)
163181
exec.removeReason = event.reason
164-
update(exec, now)
182+
update(exec, now, last = true)
165183

166184
// Remove all RDD distributions that reference the removed executor, in case there wasn't
167185
// a corresponding event.
@@ -290,8 +308,11 @@ private[spark] class AppStatusListener(
290308
}
291309

292310
job.completionTime = if (event.time > 0) Some(new Date(event.time)) else None
293-
update(job, now)
311+
update(job, now, last = true)
294312
}
313+
314+
appSummary = new AppSummary(appSummary.numCompletedJobs + 1, appSummary.numCompletedStages)
315+
kvstore.write(appSummary)
295316
}
296317

297318
override def onStageSubmitted(event: SparkListenerStageSubmitted): Unit = {
@@ -350,6 +371,13 @@ private[spark] class AppStatusListener(
350371
job.activeTasks += 1
351372
maybeUpdate(job, now)
352373
}
374+
375+
if (stage.savedTasks.incrementAndGet() > maxTasksPerStage && !stage.cleaning) {
376+
stage.cleaning = true
377+
kvstore.doAsync {
378+
cleanupTasks(stage)
379+
}
380+
}
353381
}
354382

355383
liveExecutors.get(event.taskInfo.executorId).foreach { exec =>
@@ -449,6 +477,13 @@ private[spark] class AppStatusListener(
449477
esummary.metrics.update(metricsDelta)
450478
}
451479
maybeUpdate(esummary, now)
480+
481+
if (!stage.cleaning && stage.savedTasks.get() > maxTasksPerStage) {
482+
stage.cleaning = true
483+
kvstore.doAsync {
484+
cleanupTasks(stage)
485+
}
486+
}
452487
}
453488

454489
liveExecutors.get(event.taskInfo.executorId).foreach { exec =>
@@ -516,8 +551,11 @@ private[spark] class AppStatusListener(
516551
}
517552

518553
stage.executorSummaries.values.foreach(update(_, now))
519-
update(stage, now)
554+
update(stage, now, last = true)
520555
}
556+
557+
appSummary = new AppSummary(appSummary.numCompletedJobs, appSummary.numCompletedStages + 1)
558+
kvstore.write(appSummary)
521559
}
522560

523561
override def onBlockManagerAdded(event: SparkListenerBlockManagerAdded): Unit = {
@@ -573,7 +611,7 @@ private[spark] class AppStatusListener(
573611
}
574612

575613
/** Flush all live entities' data to the underlying store. */
576-
def flush(): Unit = {
614+
private def flush(): Unit = {
577615
val now = System.nanoTime()
578616
liveStages.values.asScala.foreach { stage =>
579617
update(stage, now)
@@ -708,7 +746,10 @@ private[spark] class AppStatusListener(
708746
}
709747

710748
private def getOrCreateExecutor(executorId: String, addTime: Long): LiveExecutor = {
711-
liveExecutors.getOrElseUpdate(executorId, new LiveExecutor(executorId, addTime))
749+
liveExecutors.getOrElseUpdate(executorId, {
750+
activeExecutorCount += 1
751+
new LiveExecutor(executorId, addTime)
752+
})
712753
}
713754

714755
private def updateStreamBlock(event: SparkListenerBlockUpdated, stream: StreamBlockId): Unit = {
@@ -754,8 +795,8 @@ private[spark] class AppStatusListener(
754795
}
755796
}
756797

757-
private def update(entity: LiveEntity, now: Long): Unit = {
758-
entity.write(kvstore, now)
798+
private def update(entity: LiveEntity, now: Long, last: Boolean = false): Unit = {
799+
entity.write(kvstore, now, checkTriggers = last)
759800
}
760801

761802
/** Update a live entity only if it hasn't been updated in the last configured period. */
@@ -772,4 +813,127 @@ private[spark] class AppStatusListener(
772813
}
773814
}
774815

816+
private def cleanupExecutors(count: Long): Unit = {
817+
// Because the limit is on the number of *dead* executors, we need to calculate whether
818+
// there are actually enough dead executors to be deleted.
819+
val threshold = conf.get(MAX_RETAINED_DEAD_EXECUTORS)
820+
val dead = count - activeExecutorCount
821+
822+
if (dead > threshold) {
823+
val countToDelete = calculateNumberToRemove(dead, threshold)
824+
val toDelete = kvstore.view(classOf[ExecutorSummaryWrapper]).index("active")
825+
.max(countToDelete).first(false).last(false).asScala.toSeq
826+
toDelete.foreach { e => kvstore.delete(e.getClass(), e.info.id) }
827+
}
828+
}
829+
830+
private def cleanupJobs(count: Long): Unit = {
831+
val countToDelete = calculateNumberToRemove(count, conf.get(MAX_RETAINED_JOBS))
832+
if (countToDelete <= 0L) {
833+
return
834+
}
835+
836+
val toDelete = KVUtils.viewToSeq(kvstore.view(classOf[JobDataWrapper]),
837+
countToDelete.toInt) { j =>
838+
j.info.status != JobExecutionStatus.RUNNING && j.info.status != JobExecutionStatus.UNKNOWN
839+
}
840+
toDelete.foreach { j => kvstore.delete(j.getClass(), j.info.jobId) }
841+
}
842+
843+
private def cleanupStages(count: Long): Unit = {
844+
val countToDelete = calculateNumberToRemove(count, conf.get(MAX_RETAINED_STAGES))
845+
if (countToDelete <= 0L) {
846+
return
847+
}
848+
849+
val stages = KVUtils.viewToSeq(kvstore.view(classOf[StageDataWrapper]),
850+
countToDelete.toInt) { s =>
851+
s.info.status != v1.StageStatus.ACTIVE && s.info.status != v1.StageStatus.PENDING
852+
}
853+
854+
stages.foreach { s =>
855+
val key = s.id
856+
kvstore.delete(s.getClass(), key)
857+
858+
val execSummaries = kvstore.view(classOf[ExecutorStageSummaryWrapper])
859+
.index("stage")
860+
.first(key)
861+
.last(key)
862+
.asScala
863+
.toSeq
864+
execSummaries.foreach { e =>
865+
kvstore.delete(e.getClass(), e.id)
866+
}
867+
868+
val tasks = kvstore.view(classOf[TaskDataWrapper])
869+
.index("stage")
870+
.first(key)
871+
.last(key)
872+
.asScala
873+
874+
tasks.foreach { t =>
875+
kvstore.delete(t.getClass(), t.info.taskId)
876+
}
877+
878+
// Check whether there are remaining attempts for the same stage. If there aren't, then
879+
// also delete the RDD graph data.
880+
val remainingAttempts = kvstore.view(classOf[StageDataWrapper])
881+
.index("stageId")
882+
.first(s.stageId)
883+
.last(s.stageId)
884+
.closeableIterator()
885+
886+
val hasMoreAttempts = try {
887+
remainingAttempts.asScala.exists { other =>
888+
other.info.attemptId != s.info.attemptId
889+
}
890+
} finally {
891+
remainingAttempts.close()
892+
}
893+
894+
if (!hasMoreAttempts) {
895+
kvstore.delete(classOf[RDDOperationGraphWrapper], s.stageId)
896+
}
897+
}
898+
}
899+
900+
private def cleanupTasks(stage: LiveStage): Unit = {
901+
val countToDelete = calculateNumberToRemove(stage.savedTasks.get(), maxTasksPerStage).toInt
902+
if (countToDelete > 0) {
903+
val stageKey = Array(stage.info.stageId, stage.info.attemptId)
904+
val view = kvstore.view(classOf[TaskDataWrapper]).index("stage").first(stageKey)
905+
.last(stageKey)
906+
907+
// Try to delete finished tasks only.
908+
val toDelete = KVUtils.viewToSeq(view, countToDelete) { t =>
909+
!live || t.info.status != TaskState.RUNNING.toString()
910+
}
911+
toDelete.foreach { t => kvstore.delete(t.getClass(), t.info.taskId) }
912+
stage.savedTasks.addAndGet(-toDelete.size)
913+
914+
// If there are more running tasks than the configured limit, delete running tasks. This
915+
// should be extremely rare since the limit should generally far exceed the number of tasks
916+
// that can run in parallel.
917+
val remaining = countToDelete - toDelete.size
918+
if (remaining > 0) {
919+
val runningTasksToDelete = view.max(remaining).iterator().asScala.toList
920+
runningTasksToDelete.foreach { t => kvstore.delete(t.getClass(), t.info.taskId) }
921+
stage.savedTasks.addAndGet(-remaining)
922+
}
923+
}
924+
stage.cleaning = false
925+
}
926+
927+
/**
928+
* Remove at least (retainedSize / 10) items to reduce friction. Because tracking may be done
929+
* asynchronously, this method may return 0 in case enough items have been deleted already.
930+
*/
931+
private def calculateNumberToRemove(dataSize: Long, retainedSize: Long): Long = {
932+
if (dataSize > retainedSize) {
933+
math.max(retainedSize / 10L, dataSize - retainedSize)
934+
} else {
935+
0L
936+
}
937+
}
938+
775939
}

core/src/main/scala/org/apache/spark/status/AppStatusPlugin.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -48,7 +48,7 @@ private[spark] trait AppStatusPlugin {
4848
*/
4949
def setupListeners(
5050
conf: SparkConf,
51-
store: KVStore,
51+
store: ElementTrackingStore,
5252
addListenerFn: SparkListener => Unit,
5353
live: Boolean): Unit
5454

core/src/main/scala/org/apache/spark/status/AppStatusStore.scala

Lines changed: 5 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -330,6 +330,10 @@ private[spark] class AppStatusStore(
330330
store.read(classOf[PoolData], name)
331331
}
332332

333+
def appSummary(): AppSummary = {
334+
store.read(classOf[AppSummary], classOf[AppSummary].getName())
335+
}
336+
333337
def close(): Unit = {
334338
store.close()
335339
}
@@ -347,7 +351,7 @@ private[spark] object AppStatusStore {
347351
* @param addListenerFn Function to register a listener with a bus.
348352
*/
349353
def createLiveStore(conf: SparkConf, addListenerFn: SparkListener => Unit): AppStatusStore = {
350-
val store = new InMemoryStore()
354+
val store = new ElementTrackingStore(new InMemoryStore(), conf)
351355
val listener = new AppStatusListener(store, conf, true)
352356
addListenerFn(listener)
353357
AppStatusPlugin.loadPlugins().foreach { p =>

0 commit comments

Comments
 (0)