aboutsummaryrefslogtreecommitdiff
path: root/streaming
diff options
context:
space:
mode:
authorTathagata Das <tathagata.das1565@gmail.com>2012-09-17 14:25:48 -0700
committerTathagata Das <tathagata.das1565@gmail.com>2012-09-17 14:25:48 -0700
commit86d420478f711e0f4eccc64c238efddf030a9b0f (patch)
tree29dd56558ffc61d6aef372aeffb1517b19c3ba2d /streaming
parent0681bbc5d9e353e6400087d36a758e89d423bca3 (diff)
downloadspark-86d420478f711e0f4eccc64c238efddf030a9b0f.tar.gz
spark-86d420478f711e0f4eccc64c238efddf030a9b0f.tar.bz2
spark-86d420478f711e0f4eccc64c238efddf030a9b0f.zip
Allowed StreamingContext to be created from existing SparkContext
Diffstat (limited to 'streaming')
-rw-r--r--streaming/src/main/scala/spark/streaming/StreamingContext.scala13
1 files changed, 5 insertions, 8 deletions
diff --git a/streaming/src/main/scala/spark/streaming/StreamingContext.scala b/streaming/src/main/scala/spark/streaming/StreamingContext.scala
index cb0f9ceb15..12f3626680 100644
--- a/streaming/src/main/scala/spark/streaming/StreamingContext.scala
+++ b/streaming/src/main/scala/spark/streaming/StreamingContext.scala
@@ -21,16 +21,13 @@ import org.apache.hadoop.io.Text
import org.apache.hadoop.mapreduce.{InputFormat => NewInputFormat}
import org.apache.hadoop.mapreduce.lib.input.TextInputFormat
-class StreamingContext (
- master: String,
- frameworkName: String,
- val sparkHome: String = null,
- val jars: Seq[String] = Nil)
- extends Logging {
-
+class StreamingContext (@transient val sc: SparkContext) extends Logging {
+
+ def this(master: String, frameworkName: String, sparkHome: String = null, jars: Seq[String] = Nil) =
+ this(new SparkContext(master, frameworkName, sparkHome, jars))
+
initLogging()
- val sc = new SparkContext(master, frameworkName, sparkHome, jars)
val env = SparkEnv.get
val inputStreams = new ArrayBuffer[InputDStream[_]]()