diff options
Diffstat (limited to 'core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala')
-rw-r--r-- | core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala | 15 |
1 files changed, 10 insertions, 5 deletions
diff --git a/core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala b/core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala index ad890b4e4d..c97543f57d 100644 --- a/core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala +++ b/core/src/test/scala/org/apache/spark/scheduler/DAGSchedulerSuite.scala @@ -94,7 +94,12 @@ class DAGSchedulerSuite extends FunSuite with BeforeAndAfter with LocalSparkCont cacheLocations.clear() results.clear() mapOutputTracker = new MapOutputTrackerMaster(conf) - scheduler = new DAGScheduler(taskScheduler, mapOutputTracker, blockManagerMaster, sc.env) { + scheduler = new DAGScheduler( + taskScheduler, + sc.listenerBus, + mapOutputTracker, + blockManagerMaster, + sc.env) { override def runLocally(job: ActiveJob) { // don't bother with the thread while unit testing runLocallyWithinThread(job) @@ -422,15 +427,15 @@ class DAGSchedulerSuite extends FunSuite with BeforeAndAfter with LocalSparkCont private def assertDataStructuresEmpty = { assert(scheduler.pendingTasks.isEmpty) assert(scheduler.activeJobs.isEmpty) - assert(scheduler.failed.isEmpty) - assert(scheduler.idToActiveJob.isEmpty) + assert(scheduler.failedStages.isEmpty) + assert(scheduler.stageIdToActiveJob.isEmpty) assert(scheduler.jobIdToStageIds.isEmpty) assert(scheduler.stageIdToJobIds.isEmpty) assert(scheduler.stageIdToStage.isEmpty) assert(scheduler.stageToInfos.isEmpty) assert(scheduler.resultStageToJob.isEmpty) - assert(scheduler.running.isEmpty) + assert(scheduler.runningStages.isEmpty) assert(scheduler.shuffleToMapStage.isEmpty) - assert(scheduler.waiting.isEmpty) + assert(scheduler.waitingStages.isEmpty) } } |