Skip to content
Merged
Show file tree
Hide file tree
Changes from 1 commit
Commits
Show all changes
73 commits
Select commit Hold shift + click to select a range
e14b545
[SPARK-7977] [BUILD] Disallowing println
jonalter Jul 10, 2015
11e22b7
[SPARK-7944] [SPARK-8013] Remove most of the Spark REPL fork for Scal…
dragos Jul 10, 2015
5dd45bd
[SPARK-8958] Dynamic allocation: change cached timeout to infinity
Jul 10, 2015
db6d57f
[CORE] [MINOR] change the log level to info
chenghao-intel Jul 10, 2015
c185f3a
[SPARK-8675] Executors created by LocalBackend won't get the same cla…
coderplay Jul 10, 2015
05ac023
[HOTFIX] fix flaky test in PySpark SQL
Jul 10, 2015
0772026
[SPARK-8923] [DOCUMENTATION, MLLIB] Add @since tags to mllib.fpm
rahulpalamuttam Jul 10, 2015
fb8807c
[SPARK-7078] [SPARK-7079] Binary processing sort for Spark SQL
JoshRosen Jul 10, 2015
857e325
[SPARK-8990] [SQL] SPARK-8990 DataFrameReader.parquet() should respec…
liancheng Jul 10, 2015
b6fc0ad
add inline comment for python tests
davies Jul 11, 2015
3363088
[SPARK-8961] [SQL] Makes BaseWriterContainer.outputWriterForRow accep…
liancheng Jul 11, 2015
6e1c7e2
[SPARK-7735] [PYSPARK] Raise Exception on non-zero exit from pipe com…
megatron-me-uk Jul 11, 2015
9c50757
[SPARK-8598] [MLLIB] Implementation of 1-sample, two-sided, Kolmogoro…
Jul 11, 2015
7f6be1f
[SPARK-6487] [MLLIB] Add sequential pattern mining algorithm PrefixSp…
zhangjiajin Jul 11, 2015
0c5207c
[SPARK-8994] [ML] tiny cleanups to Params, Pipeline
jkbradley Jul 11, 2015
c472eb1
[SPARK-8970][SQL] remove unnecessary abstraction for ExtractValue
cloud-fan Jul 11, 2015
3009088
[SPARK-8880] Fix confusing Stage.attemptId member variable
kayousterhout Jul 13, 2015
20b4743
[SPARK-9006] [PYSPARK] fix microsecond loss in Python 3
Jul 13, 2015
92540d2
[SPARK-8203] [SPARK-8204] [SQL] conditional function: least/greatest
adrian-wang Jul 13, 2015
6b89943
[SPARK-8944][SQL] Support casting between IntervalType and StringType
cloud-fan Jul 13, 2015
a5bc803
[SPARK-8596] Add module for rstudio link to spark
koaning Jul 13, 2015
7f487c8
[SPARK-6797] [SPARKR] Add support for YARN cluster mode.
Jul 13, 2015
9b62e93
[SPARK-8706] [PYSPARK] [PROJECT INFRA] Add pylint checks to PySpark
MechCoder Jul 13, 2015
5ca26fb
[SPARK-8950] [WEBUI] Correct the calculation of SchedulerDelay in Sta…
carsonwang Jul 13, 2015
79c3582
Revert "[SPARK-8706] [PYSPARK] [PROJECT INFRA] Add pylint checks to P…
davies Jul 13, 2015
5c41691
[SPARK-8954] [BUILD] Remove unneeded deb repository from Dockerfile t…
yongtang Jul 13, 2015
714fc55
[SPARK-8991] [ML] Update SharedParamsCodeGen's Generated Documentation
Jul 13, 2015
4c797f2
[SPARK-8636] [SQL] Fix equalNullSafe comparison
Jul 13, 2015
0aed38e
[SPARK-8533] [STREAMING] Upgrade Flume to 1.6.0
harishreedharan Jul 13, 2015
b7bcbe2
[SPARK-8743] [STREAMING] Deregister Codahale metrics for streaming wh…
Jul 13, 2015
408b384
[SPARK-6910] [SQL] Support for pushing predicates down to metastore f…
Jul 14, 2015
20c1434
[SPARK-9001] Fixing errors in javadocs that lead to failed build/sbt doc
jegonzal Jul 14, 2015
c1feebd
[SPARK-9010] [DOCUMENTATION] Improve the Spark Configuration document…
stanzhai Jul 14, 2015
257236c
[SPARK-6851] [SQL] function least/greatest follow up
adrian-wang Jul 14, 2015
59d820a
[SPARK-9029] [SQL] shortcut CaseKeyWhen if key is null
cloud-fan Jul 14, 2015
37f2d96
[SPARK-9027] [SQL] Generalize metastore predicate pushdown
marmbrus Jul 14, 2015
c4e98ff
[SPARK-8933] [BUILD] Provide a --force flag to build/mvn that always …
Jul 14, 2015
8fb3a65
[SPARK-8911] Fix local mode endless heartbeats
Jul 14, 2015
d267c28
[SPARK-9031] Merge BlockObjectWriter and DiskBlockObject writer to re…
JoshRosen Jul 14, 2015
0a4071e
[SPARK-8718] [GRAPHX] Improve EdgePartition2D for non perfect square …
aray Jul 14, 2015
fb1d06f
[SPARK-4072] [CORE] Display Streaming blocks in Streaming UI
zsxwing Jul 14, 2015
4b5cfc9
[SPARK-8800] [SQL] Fix inaccurate precision/scale of Decimal division…
viirya Jul 14, 2015
740b034
[SPARK-4362] [MLLIB] Make prediction probability available in NaiveBa…
srowen Jul 14, 2015
11e5c37
[SPARK-8962] Add Scalastyle rule to ban direct use of Class.forName; …
JoshRosen Jul 14, 2015
e965a79
[SPARK-9045] Fix Scala 2.11 build break in UnsafeExternalRowSorter
JoshRosen Jul 15, 2015
cc57d70
[SPARK-9050] [SQL] Remove unused newOrdering argument from Exchange (…
JoshRosen Jul 15, 2015
f957796
[SPARK-8820] [STREAMING] Add a configuration to set checkpoint dir.
SaintBacchus Jul 15, 2015
bb870e7
[SPARK-5523] [CORE] [STREAMING] Add a cache for hostname in TaskMetri…
jerryshao Jul 15, 2015
5572fd0
[HOTFIX] Adding new names to known contributors
pwendell Jul 15, 2015
f650a00
[SPARK-8808] [SPARKR] Fix assignments in SparkR.
Jul 15, 2015
f23a721
[SPARK-8993][SQL] More comprehensive type checking in expressions.
rxin Jul 15, 2015
c6b1a9e
Revert SPARK-6910 and SPARK-9027
marmbrus Jul 15, 2015
4692769
[SPARK-6259] [MLLIB] Python API for LDA
yu-iskw Jul 15, 2015
3f6296f
[SPARK-8018] [MLLIB] KMeans should accept initial cluster centers as …
FlytxtRnD Jul 15, 2015
f0e1297
[SPARK-8279][SQL]Add math function round
yjshen Jul 15, 2015
1bb8acc
[SPARK-8997] [MLLIB] Performance improvements in LocalPrefixSpan
Jul 15, 2015
14935d8
[HOTFIX][SQL] Unit test breaking.
rxin Jul 15, 2015
adb33d3
[SPARK-9012] [WEBUI] Escape Accumulators in the task table
zsxwing Jul 15, 2015
20bb10f
[SPARK-8706] [PYSPARK] [PROJECT INFRA] Add pylint checks to PySpark
MechCoder Jul 15, 2015
6f69025
[SPARK-8840] [SPARKR] Add float coercion on SparkR
viirya Jul 15, 2015
fa4ec36
[SPARK-9020][SQL] Support mutable state in code gen expressions
cloud-fan Jul 15, 2015
a938527
[SPARK-8221][SQL]Add pmod function
zhichao-li Jul 15, 2015
9716a72
[Minor][SQL] Allow spaces in the beginning and ending of string for I…
viirya Jul 15, 2015
303c120
[SPARK-7555] [DOCS] Add doc for elastic net in ml-guide and mllib-guide
coderxiang Jul 15, 2015
ec9b621
SPARK-9070 JavaDataFrameSuite teardown NPEs if setup failed
steveloughran Jul 15, 2015
536533c
[SPARK-9005] [MLLIB] Fix RegressionMetrics computation of explainedVa…
Jul 15, 2015
b9a922e
[SPARK-6602][Core]Replace Akka Serialization with Spark Serializer
zsxwing Jul 15, 2015
674eb2a
[SPARK-8974] Catch exceptions in allocation schedule task.
Jul 15, 2015
affbe32
[SPARK-9071][SQL] MonotonicallyIncreasingID and SparkPartitionID shou…
rxin Jul 15, 2015
5599cc4
Predicate pushdown to hive metastore
Jul 15, 2015
b3cb5af
Synchronize getPartitionsByFilter
Jul 17, 2015
acf96d1
Synchronize on hive client
Jul 17, 2015
f897087
Synchronize on this
Jul 17, 2015
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Prev Previous commit
Next Next commit
[SPARK-8743] [STREAMING] Deregister Codahale metrics for streaming wh…
…en StreamingContext is closed

The issue link: https://issues.apache.org/jira/browse/SPARK-8743
Deregister Codahale metrics for streaming when StreamingContext is closed

Design:
Adding the method calls in the appropriate start() and stop () methods for the StreamingContext

Actions in the PullRequest:
1) Added the registerSource method call to the start method for the Streaming Context.
2) Added the removeSource method to the stop method.
3) Added comments for both 1 and 2 and comment to show initialization of the StreamingSource
4) Added a test case to check for both registration and de-registration of metrics

Previous closed PR for reference: apache#7250

Author: Neelesh Srinivas Salian <[email protected]>

Closes apache#7362 from nssalian/branch-SPARK-8743 and squashes the following commits:

7d998a3 [Neelesh Srinivas Salian] Removed the Thread.sleep() call
8b26397 [Neelesh Srinivas Salian] Moved the scalatest.{} import
0e8007a [Neelesh Srinivas Salian] moved import org.apache.spark{} to correct place
daedaa5 [Neelesh Srinivas Salian] Corrected Ordering of imports
8873180 [Neelesh Srinivas Salian] Removed redundancy in imports
59227a4 [Neelesh Srinivas Salian] Changed the ordering of the imports to classify  scala and spark imports
d8cb577 [Neelesh Srinivas Salian] Added registerSource to start() and removeSource to stop(). Wrote a test to check the registration and de-registration
  • Loading branch information
Neelesh Srinivas Salian authored and tdas committed Jul 13, 2015
commit b7bcbe25f90ba4e78b548465bc80d4de1d2c4a4a
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 @@ -20,20 +20,23 @@ package org.apache.spark.streaming
import java.io.{File, NotSerializableException}
import java.util.concurrent.atomic.AtomicInteger

import scala.collection.mutable.ArrayBuffer
import scala.collection.mutable.Queue

import org.apache.commons.io.FileUtils
import org.scalatest.{Assertions, BeforeAndAfter, PrivateMethodTester}
import org.scalatest.concurrent.Eventually._
import org.scalatest.concurrent.Timeouts
import org.scalatest.exceptions.TestFailedDueToTimeoutException
import org.scalatest.time.SpanSugar._
import org.scalatest.{Assertions, BeforeAndAfter}

import org.apache.spark.{Logging, SparkConf, SparkContext, SparkFunSuite}
import org.apache.spark.metrics.MetricsSystem
import org.apache.spark.metrics.source.Source
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}


class StreamingContextSuite extends SparkFunSuite with BeforeAndAfter with Timeouts with Logging {
Expand Down Expand Up @@ -299,6 +302,25 @@ 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)

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 +833,18 @@ 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())
}
}