From 4e5b09664cdf95effff61c042b6243107355b55c Mon Sep 17 00:00:00 2001 From: Prashant Sharma <prashant.s@imaginea.com> Date: Wed, 20 Feb 2013 12:33:37 +0530 Subject: [PATCH] fixes corresponding to review feedback at pull request #479 --- .../scala/spark/streaming/examples/ZeroMQWordCount.scala | 9 ++++++--- project/SparkBuild.scala | 4 ++-- .../main/scala/spark/streaming/StreamingContext.scala | 2 +- 3 files changed, 9 insertions(+), 6 deletions(-) diff --git a/examples/src/main/scala/spark/streaming/examples/ZeroMQWordCount.scala b/examples/src/main/scala/spark/streaming/examples/ZeroMQWordCount.scala index ab7b67ed4b..5ed9b7cb76 100644 --- a/examples/src/main/scala/spark/streaming/examples/ZeroMQWordCount.scala +++ b/examples/src/main/scala/spark/streaming/examples/ZeroMQWordCount.scala @@ -35,20 +35,23 @@ object SimpleZeroMQPublisher { /** * A sample wordcount with ZeroMQStream stream * - * Usage: WordCountZeroMQ <master> <zeroMQurl> <topic> + * To work with zeroMQ, some native libraries have to be installed. + * Install zeroMQ (release 2.1) core libraries. [ZeroMQ Install guide](http://www.zeromq.org/intro:get-the-software) + * + * Usage: ZeroMQWordCount <master> <zeroMQurl> <topic> * In local mode, <master> should be 'local[n]' with n > 1 * <zeroMQurl> and <topic> describe where zeroMq publisher is running. * * To run this example locally, you may run publisher as * `$ ./run spark.streaming.examples.SimpleZeroMQPublisher tcp://127.0.1.1:1234 foo.bar` - * and then run the example + * and run the example as * `$ ./run spark.streaming.examples.ZeroMQWordCount local[2] tcp://127.0.1.1:1234 foo` */ object ZeroMQWordCount { def main(args: Array[String]) { if (args.length < 3) { System.err.println( - "Usage: WordCountZeroMQ <master> <zeroMQurl> <topic>" + + "Usage: ZeroMQWordCount <master> <zeroMQurl> <topic>" + "In local mode, <master> should be 'local[n]' with n > 1") System.exit(1) } diff --git a/project/SparkBuild.scala b/project/SparkBuild.scala index 5fe85a28c3..7f432b60db 100644 --- a/project/SparkBuild.scala +++ b/project/SparkBuild.scala @@ -134,7 +134,6 @@ object SparkBuild extends Build { "com.typesafe.akka" % "akka-actor" % "2.0.3", "com.typesafe.akka" % "akka-remote" % "2.0.3", "com.typesafe.akka" % "akka-slf4j" % "2.0.3", - "com.typesafe.akka" % "akka-zeromq" % "2.0.3", "it.unimi.dsi" % "fastutil" % "6.4.4", "colt" % "colt" % "1.2.0", "cc.spray" % "spray-can" % "1.0-M2.1", @@ -165,7 +164,8 @@ object SparkBuild extends Build { libraryDependencies ++= Seq( "org.apache.flume" % "flume-ng-sdk" % "1.2.0" % "compile", "com.github.sgroschupf" % "zkclient" % "0.1", - "org.twitter4j" % "twitter4j-stream" % "3.0.3" + "org.twitter4j" % "twitter4j-stream" % "3.0.3", + "com.typesafe.akka" % "akka-zeromq" % "2.0.3" ) ) ++ assemblySettings ++ extraAssemblySettings diff --git a/streaming/src/main/scala/spark/streaming/StreamingContext.scala b/streaming/src/main/scala/spark/streaming/StreamingContext.scala index f15e6bd23d..2ca7dcc218 100644 --- a/streaming/src/main/scala/spark/streaming/StreamingContext.scala +++ b/streaming/src/main/scala/spark/streaming/StreamingContext.scala @@ -177,7 +177,7 @@ class StreamingContext private ( } /** - * ZeroMQ stream receiver + * Create an input stream that receives messages pushed by a zeromq publisher. * @param publisherUrl Url of remote zeromq publisher * @param zeroMQ topic to subscribe to * @param bytesToObjects A zeroMQ stream publishes sequence of frames for each topic and each frame has sequence -- GitLab