Skip to content

Commit 47ef86c

Browse files
committed
Changes based on Discussions on PR, removing use of RawTextHelper from examples
1 parent 8ed2d3f commit 47ef86c

File tree

3 files changed

+2
-7
lines changed

3 files changed

+2
-7
lines changed

examples/src/main/scala/org/apache/spark/examples/streaming/KafkaWordCount.scala

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,6 @@ import kafka.producer._
2424
import org.apache.spark.streaming._
2525
import org.apache.spark.streaming.StreamingContext._
2626
import org.apache.spark.streaming.kafka._
27-
import org.apache.spark.streaming.util.RawTextHelper._
2827

2928
// scalastyle:off
3029
/**
@@ -59,7 +58,7 @@ object KafkaWordCount {
5958
val lines = KafkaUtils.createStream(ssc, zkQuorum, group, topicpMap).map(_._2)
6059
val words = lines.flatMap(_.split(" "))
6160
val wordCounts = words.map(x => (x, 1L))
62-
.reduceByKeyAndWindow(add _, subtract _, Minutes(10), Seconds(2), 2)
61+
.reduceByKeyAndWindow(_ + _, _ - _, Minutes(10), Seconds(2), 2)
6362
wordCounts.print()
6463

6564
ssc.start()

examples/src/main/scala/org/apache/spark/examples/streaming/RawNetworkGrep.scala

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,6 @@ package org.apache.spark.examples.streaming
1919

2020
import org.apache.spark.storage.StorageLevel
2121
import org.apache.spark.streaming._
22-
import org.apache.spark.streaming.util.RawTextHelper
2322
import org.apache.spark.util.IntParam
2423

2524
/**
@@ -52,9 +51,6 @@ object RawNetworkGrep {
5251
val ssc = new StreamingContext(master, "RawNetworkGrep", Milliseconds(batchMillis),
5352
System.getenv("SPARK_HOME"), StreamingContext.jarOfClass(this.getClass).toSeq)
5453

55-
// Warm up the JVMs on master and slave for JIT compilation to kick in
56-
RawTextHelper.warmUp(ssc.sparkContext)
57-
5854
val rawStreams = (1 to numStreams).map(_ =>
5955
ssc.rawSocketStream[String](host, port, StorageLevel.MEMORY_ONLY_SER_2)).toArray
6056
val union = ssc.union(rawStreams)

streaming/src/main/scala/org/apache/spark/streaming/util/RawTextHelper.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -22,7 +22,7 @@ import org.apache.spark.SparkContext._
2222
import org.apache.spark.util.collection.OpenHashMap
2323
import scala.collection.JavaConversions.mapAsScalaMap
2424

25-
private[spark]
25+
private[streaming]
2626
object RawTextHelper {
2727

2828
/**

0 commit comments

Comments
 (0)