Skip to content

Commit 1e6e88a

Browse files
author
Sital Kedia
committed
May be fix tests?
1 parent d32856d commit 1e6e88a

File tree

1 file changed

+9
-4
lines changed

1 file changed

+9
-4
lines changed

core/src/main/scala/org/apache/spark/MapOutputTracker.scala

Lines changed: 9 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -426,11 +426,16 @@ private[spark] class MapOutputTrackerMaster(conf: SparkConf,
426426

427427
/** Get the epoch for map output for a shuffle, if it is available */
428428
def getEpochForMapOutput(shuffleId: Int, mapId: Int): Option[Long] = {
429-
for {
430-
mapStatus <- mapStatuses.get(shuffleId).flatMap { mapStatusArray =>
431-
Option(mapStatusArray(mapId))
429+
val arrayOpt = mapStatuses.get(shuffleId)
430+
if (arrayOpt.isDefined && arrayOpt.get != null) {
431+
val array = arrayOpt.get
432+
array.synchronized {
433+
if (array(mapId) != null) {
434+
return Some(epochForMapStatus(shuffleId)(mapId))
435+
}
432436
}
433-
} yield epochForMapStatus(shuffleId)(mapId)
437+
}
438+
None
434439
}
435440

436441
/**

0 commit comments

Comments
 (0)