17
17
18
18
package org .apache .spark .streaming .kafka
19
19
20
+ import java .io .File
21
+
20
22
import scala .collection .mutable
21
23
22
24
import kafka .serializer .StringDecoder
@@ -25,6 +27,7 @@ import kafka.utils.{ZkUtils, ZKGroupTopicDirs}
25
27
import org .apache .spark .SparkConf
26
28
import org .apache .spark .storage .StorageLevel
27
29
import org .apache .spark .streaming .StreamingContext
30
+ import org .apache .spark .util .Utils
28
31
29
32
class ReliableKafkaStreamSuite extends KafkaStreamSuite {
30
33
import KafkaTestUtils ._
@@ -35,6 +38,11 @@ class ReliableKafkaStreamSuite extends KafkaStreamSuite {
35
38
.setAppName(framework)
36
39
.set(" spark.streaming.receiver.writeAheadLog.enable" , " true" )
37
40
val ssc = new StreamingContext (sparkConf, batchDuration)
41
+ val checkpointDir = s " ${System .getProperty(" java.io.tmpdir" , " /tmp" )}/ " +
42
+ s " test-checkpoint ${random.nextInt(10000 )}"
43
+ Utils .registerShutdownDeleteDir(new File (checkpointDir))
44
+ ssc.checkpoint(checkpointDir)
45
+
38
46
val topic = " test"
39
47
val sent = Map (" a" -> 1 , " b" -> 1 , " c" -> 1 )
40
48
createTopic(topic)
@@ -73,6 +81,11 @@ class ReliableKafkaStreamSuite extends KafkaStreamSuite {
73
81
.setAppName(framework)
74
82
.set(" spark.streaming.receiver.writeAheadLog.enable" , " true" )
75
83
val ssc = new StreamingContext (sparkConf, batchDuration)
84
+ val checkpointDir = s " ${System .getProperty(" java.io.tmpdir" , " /tmp" )}/ " +
85
+ s " test-checkpoint ${random.nextInt(10000 )}"
86
+ Utils .registerShutdownDeleteDir(new File (checkpointDir))
87
+ ssc.checkpoint(checkpointDir)
88
+
76
89
val topic = " test"
77
90
val sent = Map (" a" -> 10 , " b" -> 10 , " c" -> 10 )
78
91
createTopic(topic)
@@ -105,6 +118,11 @@ class ReliableKafkaStreamSuite extends KafkaStreamSuite {
105
118
.setAppName(framework)
106
119
.set(" spark.streaming.receiver.writeAheadLog.enable" , " true" )
107
120
val ssc = new StreamingContext (sparkConf, batchDuration)
121
+ val checkpointDir = s " ${System .getProperty(" java.io.tmpdir" , " /tmp" )}/ " +
122
+ s " test-checkpoint ${random.nextInt(10000 )}"
123
+ Utils .registerShutdownDeleteDir(new File (checkpointDir))
124
+ ssc.checkpoint(checkpointDir)
125
+
108
126
val topics = Map (" topic1" -> 1 , " topic2" -> 1 , " topic3" -> 1 )
109
127
val sent = Map (" a" -> 10 , " b" -> 10 , " c" -> 10 )
110
128
topics.foreach { case (t, _) =>
@@ -133,61 +151,6 @@ class ReliableKafkaStreamSuite extends KafkaStreamSuite {
133
151
topics.foreach { case (t, _) => assert(getCommitOffset(groupId, t, 0 ) === 29L ) }
134
152
}
135
153
136
- test(" Verify offset commit when exception is met" ) {
137
- val sparkConf = new SparkConf ()
138
- .setMaster(master)
139
- .setAppName(framework)
140
- .set(" spark.streaming.receiver.writeAheadLog.enable" , " true" )
141
- var ssc = new StreamingContext (
142
- sparkConf.clone.set(" spark.streaming.blockInterval" , " 10000" ),
143
- batchDuration)
144
- val topics = Map (" topic1" -> 1 , " topic2" -> 1 , " topic3" -> 1 )
145
- val sent = Map (" a" -> 10 , " b" -> 10 , " c" -> 10 )
146
- topics.foreach { case (t, _) =>
147
- createTopic(t)
148
- produceAndSendMessage(t, sent)
149
- }
150
-
151
- val groupId = s " test-consumer- ${random.nextInt(10000 )}"
152
-
153
- val kafkaParams = Map (" zookeeper.connect" -> s " $zkHost: $zkPort" ,
154
- " group.id" -> groupId,
155
- " auto.offset.reset" -> " smallest" )
156
-
157
- KafkaUtils .createStream[String , String , StringDecoder , StringDecoder ](
158
- ssc,
159
- kafkaParams,
160
- topics,
161
- StorageLevel .MEMORY_ONLY )
162
- .foreachRDD(_ => throw new Exception )
163
- try {
164
- ssc.start()
165
- ssc.awaitTermination(1000 )
166
- } catch {
167
- case e : Exception =>
168
- if (ssc != null ) {
169
- ssc.stop()
170
- ssc = null
171
- }
172
- }
173
- // Failed before putting to BM, so offset is not updated.
174
- topics.foreach { case (t, _) => assert(getCommitOffset(groupId, t, 0 ) === 0L ) }
175
-
176
- // Restart to see if data is consumed from last checkpoint.
177
- ssc = new StreamingContext (sparkConf, batchDuration)
178
- KafkaUtils .createStream[String , String , StringDecoder , StringDecoder ](
179
- ssc,
180
- kafkaParams,
181
- topics,
182
- StorageLevel .MEMORY_ONLY )
183
- .foreachRDD(_ => Unit )
184
- ssc.start()
185
- ssc.awaitTermination(3000 )
186
- ssc.stop()
187
-
188
- topics.foreach { case (t, _) => assert(getCommitOffset(groupId, t, 0 ) === 29L ) }
189
- }
190
-
191
154
private def getCommitOffset (groupId : String , topic : String , partition : Int ): Long = {
192
155
assert(zkClient != null , " Zookeeper client is not initialized" )
193
156
0 commit comments