-
Notifications
You must be signed in to change notification settings - Fork 13k
/
Kafka010ITCase.java
330 lines (266 loc) · 10.1 KB
/
Kafka010ITCase.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.connectors.kafka;
import org.apache.flink.api.common.ExecutionConfig;
import org.apache.flink.api.common.restartstrategy.RestartStrategies;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.common.typeutils.TypeSerializer;
import org.apache.flink.api.java.typeutils.GenericTypeInfo;
import org.apache.flink.api.java.typeutils.TypeInfoParser;
import org.apache.flink.core.memory.DataInputView;
import org.apache.flink.core.memory.DataInputViewStreamWrapper;
import org.apache.flink.streaming.api.TimeCharacteristic;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.AssignerWithPunctuatedWatermarks;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.api.operators.StreamSink;
import org.apache.flink.streaming.api.watermark.Watermark;
import org.apache.flink.streaming.connectors.kafka.partitioner.KafkaPartitioner;
import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.streaming.util.serialization.KeyedDeserializationSchema;
import org.apache.flink.streaming.util.serialization.KeyedSerializationSchemaWrapper;
import org.apache.flink.streaming.util.serialization.TypeInformationSerializationSchema;
import org.junit.Test;
import javax.annotation.Nullable;
import java.io.ByteArrayInputStream;
import java.io.IOException;
public class Kafka010ITCase extends KafkaConsumerTestBase {
// ------------------------------------------------------------------------
// Suite of Tests
// ------------------------------------------------------------------------
@Test(timeout = 60000)
public void testFailOnNoBroker() throws Exception {
runFailOnNoBrokerTest();
}
@Test(timeout = 60000)
public void testConcurrentProducerConsumerTopology() throws Exception {
runSimpleConcurrentProducerConsumerTopology();
}
@Test(timeout = 60000)
public void testKeyValueSupport() throws Exception {
runKeyValueTest();
}
// --- canceling / failures ---
@Test(timeout = 60000)
public void testCancelingEmptyTopic() throws Exception {
runCancelingOnEmptyInputTest();
}
@Test(timeout = 60000)
public void testCancelingFullTopic() throws Exception {
runCancelingOnFullInputTest();
}
@Test(timeout = 60000)
public void testFailOnDeploy() throws Exception {
runFailOnDeployTest();
}
// --- source to partition mappings and exactly once ---
@Test(timeout = 60000)
public void testOneToOneSources() throws Exception {
runOneToOneExactlyOnceTest();
}
@Test(timeout = 60000)
public void testOneSourceMultiplePartitions() throws Exception {
runOneSourceMultiplePartitionsExactlyOnceTest();
}
@Test(timeout = 60000)
public void testMultipleSourcesOnePartition() throws Exception {
runMultipleSourcesOnePartitionExactlyOnceTest();
}
// --- broker failure ---
@Test(timeout = 60000)
public void testBrokerFailure() throws Exception {
runBrokerFailureTest();
}
// --- special executions ---
@Test(timeout = 60000)
public void testBigRecordJob() throws Exception {
runBigRecordTestTopology();
}
@Test(timeout = 60000)
public void testMultipleTopics() throws Exception {
runProduceConsumeMultipleTopics();
}
@Test(timeout = 60000)
public void testAllDeletes() throws Exception {
runAllDeletesTest();
}
@Test(timeout = 60000)
public void testMetricsAndEndOfStream() throws Exception {
runEndOfStreamTest();
}
// --- startup mode ---
@Test(timeout = 60000)
public void testStartFromEarliestOffsets() throws Exception {
runStartFromEarliestOffsets();
}
@Test(timeout = 60000)
public void testStartFromLatestOffsets() throws Exception {
runStartFromLatestOffsets();
}
@Test(timeout = 60000)
public void testStartFromGroupOffsets() throws Exception {
runStartFromGroupOffsets();
}
// --- offset committing ---
@Test(timeout = 60000)
public void testCommitOffsetsToKafka() throws Exception {
runCommitOffsetsToKafka();
}
@Test(timeout = 60000)
public void testStartFromKafkaCommitOffsets() throws Exception {
runStartFromKafkaCommitOffsets();
}
@Test(timeout = 60000)
public void testAutoOffsetRetrievalAndCommitToKafka() throws Exception {
runAutoOffsetRetrievalAndCommitToKafka();
}
/**
* Kafka 0.10 specific test, ensuring Timestamps are properly written to and read from Kafka
*/
@Test(timeout = 60000)
public void testTimestamps() throws Exception {
final String topic = "tstopic";
createTestTopic(topic, 3, 1);
// ---------- Produce an event time stream into Kafka -------------------
StreamExecutionEnvironment env = StreamExecutionEnvironment.createRemoteEnvironment("localhost", flinkPort);
env.setParallelism(1);
env.getConfig().setRestartStrategy(RestartStrategies.noRestart());
env.getConfig().disableSysoutLogging();
env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
DataStream<Long> streamWithTimestamps = env.addSource(new SourceFunction<Long>() {
boolean running = true;
@Override
public void run(SourceContext<Long> ctx) throws Exception {
long i = 0;
while(running) {
ctx.collectWithTimestamp(i, i*2);
if(i++ == 1000L) {
running = false;
}
}
}
@Override
public void cancel() {
running = false;
}
});
final TypeInformationSerializationSchema<Long> longSer = new TypeInformationSerializationSchema<>(TypeInfoParser.<Long>parse("Long"), env.getConfig());
FlinkKafkaProducer010.FlinkKafkaProducer010Configuration prod = FlinkKafkaProducer010.writeToKafkaWithTimestamps(streamWithTimestamps, topic, new KeyedSerializationSchemaWrapper<>(longSer), standardProps, new KafkaPartitioner<Long>() {
@Override
public int partition(Long next, byte[] serializedKey, byte[] serializedValue, int numPartitions) {
return (int)(next % 3);
}
});
prod.setParallelism(3);
prod.setWriteTimestampToKafka(true);
env.execute("Produce some");
// ---------- Consume stream from Kafka -------------------
env = StreamExecutionEnvironment.createRemoteEnvironment("localhost", flinkPort);
env.setParallelism(1);
env.getConfig().setRestartStrategy(RestartStrategies.noRestart());
env.getConfig().disableSysoutLogging();
env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
FlinkKafkaConsumer010<Long> kafkaSource = new FlinkKafkaConsumer010<>(topic, new LimitedLongDeserializer(), standardProps);
kafkaSource.assignTimestampsAndWatermarks(new AssignerWithPunctuatedWatermarks<Long>() {
@Nullable
@Override
public Watermark checkAndGetNextWatermark(Long lastElement, long extractedTimestamp) {
if(lastElement % 10 == 0) {
return new Watermark(lastElement);
}
return null;
}
@Override
public long extractTimestamp(Long element, long previousElementTimestamp) {
return previousElementTimestamp;
}
});
DataStream<Long> stream = env.addSource(kafkaSource);
GenericTypeInfo<Object> objectTypeInfo = new GenericTypeInfo<>(Object.class);
stream.transform("timestamp validating operator", objectTypeInfo, new TimestampValidatingOperator()).setParallelism(1);
env.execute("Consume again");
deleteTestTopic(topic);
}
private static class TimestampValidatingOperator extends StreamSink<Long> {
public TimestampValidatingOperator() {
super(new SinkFunction<Long>() {
@Override
public void invoke(Long value) throws Exception {
throw new RuntimeException("Unexpected");
}
});
}
long elCount = 0;
long wmCount = 0;
long lastWM = Long.MIN_VALUE;
@Override
public void processElement(StreamRecord<Long> element) throws Exception {
elCount++;
if(element.getValue() * 2 != element.getTimestamp()) {
throw new RuntimeException("Invalid timestamp: " + element);
}
}
@Override
public void processWatermark(Watermark mark) throws Exception {
wmCount++;
if(lastWM <= mark.getTimestamp()) {
lastWM = mark.getTimestamp();
} else {
throw new RuntimeException("Received watermark higher than the last one");
}
if( mark.getTimestamp() % 10 != 0 && mark.getTimestamp() != Long.MAX_VALUE ) {
throw new RuntimeException("Invalid watermark: " + mark.getTimestamp());
}
}
@Override
public void close() throws Exception {
super.close();
if(elCount != 1000L) {
throw new RuntimeException("Wrong final element count " + elCount);
}
if(wmCount <= 2) {
throw new RuntimeException("Almost no watermarks have been sent " + wmCount);
}
}
}
private static class LimitedLongDeserializer implements KeyedDeserializationSchema<Long> {
private final TypeInformation<Long> ti;
private final TypeSerializer<Long> ser;
long cnt = 0;
public LimitedLongDeserializer() {
this.ti = TypeInfoParser.parse("Long");
this.ser = ti.createSerializer(new ExecutionConfig());
}
@Override
public TypeInformation<Long> getProducedType() {
return ti;
}
@Override
public Long deserialize(byte[] messageKey, byte[] message, String topic, int partition, long offset) throws IOException {
cnt++;
DataInputView in = new DataInputViewStreamWrapper(new ByteArrayInputStream(message));
Long e = ser.deserialize(in);
return e;
}
@Override
public boolean isEndOfStream(Long nextElement) {
return cnt > 1000L;
}
}
}