Skip to content

Commit

Permalink
Added registerSource to start() and removeSource to stop(). Wrote a t…
Browse files Browse the repository at this point in the history
…est to check the registration and de-registration
  • Loading branch information
Neelesh Srinivas Salian committed Jul 13, 2015
1 parent c472eb1 commit d8cb577
Show file tree
Hide file tree
Showing 2 changed files with 48 additions and 6 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -192,11 +192,8 @@ class StreamingContext private[streaming] (
None
}

/** Register streaming source to metrics system */
/* Initializing a streamingSource to register metrics */
private val streamingSource = new StreamingSource(this)
assert(env != null)
assert(env.metricsSystem != null)
env.metricsSystem.registerSource(streamingSource)

private var state: StreamingContextState = INITIALIZED

Expand Down Expand Up @@ -606,6 +603,9 @@ class StreamingContext private[streaming] (
}
shutdownHookRef = Utils.addShutdownHook(
StreamingContext.SHUTDOWN_HOOK_PRIORITY)(stopOnShutdown)
// Registering Streaming Metrics at the start of the StreamingContext
assert(env.metricsSystem != null)
env.metricsSystem.registerSource(streamingSource)
uiTab.foreach(_.attach())
logInfo("StreamingContext started")
case ACTIVE =>
Expand Down Expand Up @@ -682,6 +682,8 @@ class StreamingContext private[streaming] (
logWarning("StreamingContext has already been stopped")
case ACTIVE =>
scheduler.stop(stopGracefully)
// Removing the streamingSource to de-register the metrics on stop()
env.metricsSystem.removeSource(streamingSource)
uiTab.foreach(_.detach())
StreamingContext.setActiveContext(null)
waiter.notifyStop()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,8 +33,12 @@ import org.apache.spark.storage.StorageLevel
import org.apache.spark.streaming.dstream.DStream
import org.apache.spark.streaming.receiver.Receiver
import org.apache.spark.util.Utils
import org.apache.spark.{Logging, SparkConf, SparkContext, SparkException, SparkFunSuite}

import org.apache.spark.{Logging, SparkConf, SparkContext, SparkFunSuite}
import org.apache.spark.metrics.MetricsSystem
import org.apache.spark.metrics.source.Source
import org.scalatest.{PrivateMethodTester, Assertions, BeforeAndAfter}
import org.apache.spark.{Logging, SparkConf, SparkContext, SparkFunSuite}
import scala.collection.mutable.ArrayBuffer

class StreamingContextSuite extends SparkFunSuite with BeforeAndAfter with Timeouts with Logging {

Expand Down Expand Up @@ -299,6 +303,26 @@ class StreamingContextSuite extends SparkFunSuite with BeforeAndAfter with Timeo
Thread.sleep(100)
}

test ("registering and de-registering of streamingSource") {
val conf = new SparkConf().setMaster(master).setAppName(appName)
ssc = new StreamingContext(conf, batchDuration)
assert(ssc.getState() === StreamingContextState.INITIALIZED)
addInputStream(ssc).register()
ssc.start()

val sources = StreamingContextSuite.getSources(ssc.env.metricsSystem)
val streamingSource = StreamingContextSuite.getStreamingSource(ssc)
assert(sources.contains(streamingSource))
assert(ssc.getState() === StreamingContextState.ACTIVE)
Thread.sleep(100)

ssc.stop()
val sourcesAfterStop = StreamingContextSuite.getSources(ssc.env.metricsSystem)
val streamingSourceAfterStop = StreamingContextSuite.getStreamingSource(ssc)
assert(ssc.getState() === StreamingContextState.STOPPED)
assert(!sourcesAfterStop.contains(streamingSourceAfterStop))
}

test("awaitTermination") {
ssc = new StreamingContext(master, appName, batchDuration)
val inputStream = addInputStream(ssc)
Expand Down Expand Up @@ -811,3 +835,19 @@ package object testPackage extends Assertions {
}
}
}

/**
* Helper methods for testing StreamingContextSuite
* This includes methods to access private methods and fields in StreamingContext and MetricsSystem
*/

private object StreamingContextSuite extends PrivateMethodTester {
private val _sources = PrivateMethod[ArrayBuffer[Source]]('sources)
private def getSources(metricsSystem: MetricsSystem): ArrayBuffer[Source] = {
metricsSystem.invokePrivate(_sources())
}
private val _streamingSource = PrivateMethod[StreamingSource]('streamingSource)
private def getStreamingSource(streamingContext: StreamingContext): StreamingSource = {
streamingContext.invokePrivate(_streamingSource())
}
}

0 comments on commit d8cb577

Please # to comment.