Skip to content

Commit

Permalink
Fix streaming test failures
Browse files Browse the repository at this point in the history
  • Loading branch information
andrewor14 committed Sep 11, 2014
1 parent 8f5ae53 commit a431b84
Show file tree
Hide file tree
Showing 3 changed files with 11 additions and 4 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -37,7 +37,7 @@ import org.apache.spark.storage.StorageLevel
import org.apache.spark.streaming.dstream._
import org.apache.spark.streaming.receiver.{ActorSupervisorStrategy, ActorReceiver, Receiver}
import org.apache.spark.streaming.scheduler._
import org.apache.spark.streaming.ui.StreamingTab
import org.apache.spark.streaming.ui.{StreamingJobProgressListener, StreamingTab}
import org.apache.spark.util.MetadataCleaner

/**
Expand Down Expand Up @@ -158,7 +158,14 @@ class StreamingContext private[streaming] (

private[streaming] val waiter = new ContextWaiter

private[streaming] val uiTab = new StreamingTab(this)
private[streaming] val progressListener = new StreamingJobProgressListener(this)

private[streaming] val uiTab: Option[StreamingTab] =
if (conf.getBoolean("spark.ui.enabled", true)) {
Some(new StreamingTab(this))
} else {
None
}

/** Register streaming source to metrics system */
private val streamingSource = new StreamingSource(this)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,7 @@ private[streaming] class StreamingSource(ssc: StreamingContext) extends Source {
override val metricRegistry = new MetricRegistry
override val sourceName = "%s.StreamingMetrics".format(ssc.sparkContext.appName)

private val streamingListener = ssc.uiTab.listener
private val streamingListener = ssc.progressListener

private def registerGauge[T](name: String, f: StreamingJobProgressListener => T,
defaultValue: T) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,7 @@ private[spark] class StreamingTab(ssc: StreamingContext)
extends SparkUITab(getSparkUI(ssc), "streaming") with Logging {

val parent = getSparkUI(ssc)
val listener = new StreamingJobProgressListener(ssc)
val listener = ssc.progressListener

ssc.addStreamingListener(listener)
attachPage(new StreamingPage(this))
Expand Down

0 comments on commit a431b84

Please sign in to comment.