Browse Source
- fixed leftJoin -> outerJoin test bug - simplified to only use values - fixed inner KTable-KTable join - fixed left KTable-KTable join - fixed outer KTable-KTable join - fixed inner, left, and outer left KStream-KStream joins - added inner KStream-KTable join - fixed left KStream-KTable join Author: Matthias J. Sax <matthias@confluent.io> Reviewers: Damian Guy <damian.guy@gmail.com>, Guozhang Wang <wangguoz@gmail.com> Closes #1777 from mjsax/kafka-4001-joinspull/1777/merge
Matthias J. Sax
8 years ago
committed by
Guozhang Wang
20 changed files with 1075 additions and 494 deletions
@ -0,0 +1,75 @@
@@ -0,0 +1,75 @@
|
||||
/** |
||||
* 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 |
||||
* <p> |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* <p> |
||||
* 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.kafka.streams.kstream.internals; |
||||
|
||||
import org.apache.kafka.streams.kstream.ValueJoiner; |
||||
import org.apache.kafka.streams.processor.AbstractProcessor; |
||||
import org.apache.kafka.streams.processor.Processor; |
||||
import org.apache.kafka.streams.processor.ProcessorContext; |
||||
import org.apache.kafka.streams.processor.ProcessorSupplier; |
||||
|
||||
class KStreamKTableJoin<K, R, V1, V2> implements ProcessorSupplier<K, V1> { |
||||
|
||||
private final KTableValueGetterSupplier<K, V2> valueGetterSupplier; |
||||
private final ValueJoiner<V1, V2, R> joiner; |
||||
private final boolean leftJoin; |
||||
|
||||
KStreamKTableJoin(final KTableImpl<K, ?, V2> table, final ValueJoiner<V1, V2, R> joiner, final boolean leftJoin) { |
||||
valueGetterSupplier = table.valueGetterSupplier(); |
||||
this.joiner = joiner; |
||||
this.leftJoin = leftJoin; |
||||
} |
||||
|
||||
@Override |
||||
public Processor<K, V1> get() { |
||||
return new KStreamKTableJoinProcessor(valueGetterSupplier.get(), leftJoin); |
||||
} |
||||
|
||||
private class KStreamKTableJoinProcessor extends AbstractProcessor<K, V1> { |
||||
|
||||
private final KTableValueGetter<K, V2> valueGetter; |
||||
private final boolean leftJoin; |
||||
|
||||
KStreamKTableJoinProcessor(final KTableValueGetter<K, V2> valueGetter, final boolean leftJoin) { |
||||
this.valueGetter = valueGetter; |
||||
this.leftJoin = leftJoin; |
||||
} |
||||
|
||||
@Override |
||||
public void init(final ProcessorContext context) { |
||||
super.init(context); |
||||
valueGetter.init(context); |
||||
} |
||||
|
||||
@Override |
||||
public void process(final K key, final V1 value) { |
||||
// we do join iff keys are equal, thus, if key is null we cannot join and just ignore the record
|
||||
//
|
||||
// we also ignore the record if value is null, because in a key-value data model a null-value indicates
|
||||
// an empty message (ie, there is nothing to be joined) -- this contrast SQL NULL semantics
|
||||
// furthermore, on left/outer joins 'null' in ValueJoiner#apply() indicates a missing record --
|
||||
// thus, to be consistent and to avoid ambiguous null semantics, null values are ignored
|
||||
if (key != null && value != null) { |
||||
final V2 value2 = valueGetter.get(key); |
||||
if (leftJoin || value2 != null) { |
||||
context().forward(key, joiner.apply(value, value2)); |
||||
} |
||||
} |
||||
} |
||||
} |
||||
} |
@ -1,66 +0,0 @@
@@ -1,66 +0,0 @@
|
||||
/** |
||||
* 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.kafka.streams.kstream.internals; |
||||
|
||||
import org.apache.kafka.streams.kstream.ValueJoiner; |
||||
import org.apache.kafka.streams.processor.AbstractProcessor; |
||||
import org.apache.kafka.streams.processor.Processor; |
||||
import org.apache.kafka.streams.processor.ProcessorContext; |
||||
import org.apache.kafka.streams.processor.ProcessorSupplier; |
||||
|
||||
class KStreamKTableLeftJoin<K, R, V1, V2> implements ProcessorSupplier<K, V1> { |
||||
|
||||
private final KTableValueGetterSupplier<K, V2> valueGetterSupplier; |
||||
private final ValueJoiner<V1, V2, R> joiner; |
||||
|
||||
KStreamKTableLeftJoin(KTableImpl<K, ?, V2> table, ValueJoiner<V1, V2, R> joiner) { |
||||
this.valueGetterSupplier = table.valueGetterSupplier(); |
||||
this.joiner = joiner; |
||||
} |
||||
|
||||
@Override |
||||
public Processor<K, V1> get() { |
||||
return new KStreamKTableLeftJoinProcessor(valueGetterSupplier.get()); |
||||
} |
||||
|
||||
private class KStreamKTableLeftJoinProcessor extends AbstractProcessor<K, V1> { |
||||
|
||||
private final KTableValueGetter<K, V2> valueGetter; |
||||
|
||||
public KStreamKTableLeftJoinProcessor(KTableValueGetter<K, V2> valueGetter) { |
||||
this.valueGetter = valueGetter; |
||||
} |
||||
|
||||
@SuppressWarnings("unchecked") |
||||
@Override |
||||
public void init(ProcessorContext context) { |
||||
super.init(context); |
||||
valueGetter.init(context); |
||||
} |
||||
|
||||
@Override |
||||
public void process(K key, V1 value) { |
||||
// if the key is null, we do not need proceed joining
|
||||
// the record with the table
|
||||
if (key != null) { |
||||
context().forward(key, joiner.apply(value, valueGetter.get(key))); |
||||
} |
||||
} |
||||
} |
||||
|
||||
} |
@ -0,0 +1,433 @@
@@ -0,0 +1,433 @@
|
||||
/** |
||||
* 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 |
||||
* <p> |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* <p> |
||||
* 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.kafka.streams.integration; |
||||
|
||||
import kafka.utils.ZkUtils; |
||||
import org.apache.kafka.clients.consumer.ConsumerConfig; |
||||
import org.apache.kafka.clients.producer.ProducerConfig; |
||||
import org.apache.kafka.common.security.JaasUtils; |
||||
import org.apache.kafka.common.serialization.LongDeserializer; |
||||
import org.apache.kafka.common.serialization.LongSerializer; |
||||
import org.apache.kafka.common.serialization.Serdes; |
||||
import org.apache.kafka.common.serialization.StringDeserializer; |
||||
import org.apache.kafka.common.serialization.StringSerializer; |
||||
import org.apache.kafka.streams.KafkaStreams; |
||||
import org.apache.kafka.streams.KeyValue; |
||||
import org.apache.kafka.streams.StreamsConfig; |
||||
import org.apache.kafka.streams.integration.utils.EmbeddedKafkaCluster; |
||||
import org.apache.kafka.streams.integration.utils.IntegrationTestUtils; |
||||
import org.apache.kafka.streams.kstream.JoinWindows; |
||||
import org.apache.kafka.streams.kstream.KStream; |
||||
import org.apache.kafka.streams.kstream.KStreamBuilder; |
||||
import org.apache.kafka.streams.kstream.KTable; |
||||
import org.apache.kafka.streams.kstream.ValueJoiner; |
||||
import org.apache.kafka.test.TestCondition; |
||||
import org.apache.kafka.test.TestUtils; |
||||
import org.junit.After; |
||||
import org.junit.AfterClass; |
||||
import org.junit.Before; |
||||
import org.junit.BeforeClass; |
||||
import org.junit.ClassRule; |
||||
import org.junit.Test; |
||||
|
||||
import java.util.Arrays; |
||||
import java.util.Collections; |
||||
import java.util.HashSet; |
||||
import java.util.Iterator; |
||||
import java.util.List; |
||||
import java.util.Properties; |
||||
import java.util.Set; |
||||
|
||||
import static org.hamcrest.MatcherAssert.assertThat; |
||||
import static org.hamcrest.core.Is.is; |
||||
|
||||
/** |
||||
* Tests all available joins of Kafka Streams DSL. |
||||
*/ |
||||
public class JoinIntegrationTest { |
||||
@ClassRule |
||||
public static final EmbeddedKafkaCluster CLUSTER = new EmbeddedKafkaCluster(1); |
||||
|
||||
private static ZkUtils zkUtils = null; |
||||
|
||||
private static final String APP_ID = "join-integration-test"; |
||||
private static final String INPUT_TOPIC_1 = "inputTopicLeft"; |
||||
private static final String INPUT_TOPIC_2 = "inputTopicRight"; |
||||
private static final String OUTPUT_TOPIC = "outputTopic"; |
||||
|
||||
private final static Properties PRODUCER_CONFIG = new Properties(); |
||||
private final static Properties RESULT_CONSUMER_CONFIG = new Properties(); |
||||
private final static Properties STREAMS_CONFIG = new Properties(); |
||||
|
||||
private KStreamBuilder builder; |
||||
private KStream<Long, String> leftStream; |
||||
private KStream<Long, String> rightStream; |
||||
private KTable<Long, String> leftTable; |
||||
private KTable<Long, String> rightTable; |
||||
|
||||
private final List<Input<String>> input = Arrays.asList( |
||||
new Input<>(INPUT_TOPIC_1, (String) null), |
||||
new Input<>(INPUT_TOPIC_2, (String) null), |
||||
new Input<>(INPUT_TOPIC_1, "A"), |
||||
new Input<>(INPUT_TOPIC_2, "a"), |
||||
new Input<>(INPUT_TOPIC_1, "B"), |
||||
new Input<>(INPUT_TOPIC_2, "b"), |
||||
new Input<>(INPUT_TOPIC_1, (String) null), |
||||
new Input<>(INPUT_TOPIC_2, (String) null), |
||||
new Input<>(INPUT_TOPIC_1, "C"), |
||||
new Input<>(INPUT_TOPIC_2, "c"), |
||||
new Input<>(INPUT_TOPIC_2, (String) null), |
||||
new Input<>(INPUT_TOPIC_1, (String) null), |
||||
new Input<>(INPUT_TOPIC_2, (String) null), |
||||
new Input<>(INPUT_TOPIC_2, "d"), |
||||
new Input<>(INPUT_TOPIC_1, "D") |
||||
); |
||||
|
||||
private final ValueJoiner<String, String, String> valueJoiner = new ValueJoiner<String, String, String>() { |
||||
@Override |
||||
public String apply(final String value1, final String value2) { |
||||
return value1 + "-" + value2; |
||||
} |
||||
}; |
||||
|
||||
private final TestCondition topicsGotDeleted = new TopicsGotDeletedCondition(); |
||||
|
||||
@BeforeClass |
||||
public static void setupConfigsAndUtils() throws Exception { |
||||
PRODUCER_CONFIG.put(ProducerConfig.BOOTSTRAP_SERVERS_CONFIG, CLUSTER.bootstrapServers()); |
||||
PRODUCER_CONFIG.put(ProducerConfig.ACKS_CONFIG, "all"); |
||||
PRODUCER_CONFIG.put(ProducerConfig.RETRIES_CONFIG, 0); |
||||
PRODUCER_CONFIG.put(ProducerConfig.KEY_SERIALIZER_CLASS_CONFIG, LongSerializer.class); |
||||
PRODUCER_CONFIG.put(ProducerConfig.VALUE_SERIALIZER_CLASS_CONFIG, StringSerializer.class); |
||||
|
||||
RESULT_CONSUMER_CONFIG.put(ConsumerConfig.BOOTSTRAP_SERVERS_CONFIG, CLUSTER.bootstrapServers()); |
||||
RESULT_CONSUMER_CONFIG.put(ConsumerConfig.GROUP_ID_CONFIG, APP_ID + "-result-consumer"); |
||||
RESULT_CONSUMER_CONFIG.put(ConsumerConfig.AUTO_OFFSET_RESET_CONFIG, "earliest"); |
||||
RESULT_CONSUMER_CONFIG.put(ConsumerConfig.KEY_DESERIALIZER_CLASS_CONFIG, LongDeserializer.class); |
||||
RESULT_CONSUMER_CONFIG.put(ConsumerConfig.VALUE_DESERIALIZER_CLASS_CONFIG, StringDeserializer.class); |
||||
|
||||
STREAMS_CONFIG.put(StreamsConfig.BOOTSTRAP_SERVERS_CONFIG, CLUSTER.bootstrapServers()); |
||||
STREAMS_CONFIG.put(StreamsConfig.ZOOKEEPER_CONNECT_CONFIG, CLUSTER.zKConnectString()); |
||||
STREAMS_CONFIG.put(ConsumerConfig.AUTO_OFFSET_RESET_CONFIG, "earliest"); |
||||
STREAMS_CONFIG.put(StreamsConfig.STATE_DIR_CONFIG, TestUtils.tempDirectory().getPath()); |
||||
STREAMS_CONFIG.put(StreamsConfig.KEY_SERDE_CLASS_CONFIG, Serdes.Long().getClass()); |
||||
STREAMS_CONFIG.put(StreamsConfig.VALUE_SERDE_CLASS_CONFIG, Serdes.String().getClass()); |
||||
STREAMS_CONFIG.put(StreamsConfig.CACHE_MAX_BYTES_BUFFERING_CONFIG, 0); |
||||
|
||||
zkUtils = ZkUtils.apply(CLUSTER.zKConnectString(), |
||||
30000, |
||||
30000, |
||||
JaasUtils.isZkSecurityEnabled()); |
||||
} |
||||
|
||||
@AfterClass |
||||
public static void release() { |
||||
if (zkUtils != null) { |
||||
zkUtils.close(); |
||||
} |
||||
} |
||||
|
||||
@Before |
||||
public void prepareTopology() throws Exception { |
||||
CLUSTER.createTopic(INPUT_TOPIC_1); |
||||
CLUSTER.createTopic(INPUT_TOPIC_2); |
||||
CLUSTER.createTopic(OUTPUT_TOPIC); |
||||
|
||||
builder = new KStreamBuilder(); |
||||
leftTable = builder.table(INPUT_TOPIC_1, "leftTable"); |
||||
rightTable = builder.table(INPUT_TOPIC_2, "rightTable"); |
||||
leftStream = leftTable.toStream(); |
||||
rightStream = rightTable.toStream(); |
||||
} |
||||
|
||||
@After |
||||
public void cleanup() throws Exception { |
||||
CLUSTER.deleteTopic(INPUT_TOPIC_1); |
||||
CLUSTER.deleteTopic(INPUT_TOPIC_2); |
||||
CLUSTER.deleteTopic(OUTPUT_TOPIC); |
||||
|
||||
TestUtils.waitForCondition(topicsGotDeleted, 120000, "Topics not deleted after 120 seconds."); |
||||
} |
||||
|
||||
private void checkResult(final String outputTopic, final List<String> expectedResult) throws Exception { |
||||
if (expectedResult != null) { |
||||
final List<String> result = IntegrationTestUtils.waitUntilMinValuesRecordsReceived(RESULT_CONSUMER_CONFIG, outputTopic, expectedResult.size(), Long.MAX_VALUE); |
||||
assertThat(result, is(expectedResult)); |
||||
} |
||||
} |
||||
|
||||
/* |
||||
* Runs the actual test. Checks the result after each input record to ensure fixed processing order. |
||||
* If an input tuple does not trigger any result, "expectedResult" should contain a "null" entry |
||||
*/ |
||||
private void runTest(final List<List<String>> expectedResult) throws Exception { |
||||
assert expectedResult.size() == input.size(); |
||||
|
||||
IntegrationTestUtils.purgeLocalStreamsState(STREAMS_CONFIG); |
||||
final KafkaStreams streams = new KafkaStreams(builder, STREAMS_CONFIG); |
||||
try { |
||||
streams.start(); |
||||
|
||||
long ts = System.currentTimeMillis(); |
||||
|
||||
final Iterator<List<String>> resultIterator = expectedResult.iterator(); |
||||
for (final Input<String> singleInput : input) { |
||||
IntegrationTestUtils.produceKeyValuesSynchronouslyWithTimestamp(singleInput.topic, Collections.singleton(singleInput.record), PRODUCER_CONFIG, ++ts); |
||||
checkResult(OUTPUT_TOPIC, resultIterator.next()); |
||||
} |
||||
} finally { |
||||
streams.close(); |
||||
} |
||||
} |
||||
|
||||
@Test |
||||
public void testInnerKStreamKStream() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-inner-KStream-KStream"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Arrays.asList("A-b", "B-b"), |
||||
null, |
||||
null, |
||||
Arrays.asList("C-a", "C-b"), |
||||
Arrays.asList("A-c", "B-c", "C-c"), |
||||
null, |
||||
null, |
||||
null, |
||||
Arrays.asList("A-d", "B-d", "C-d"), |
||||
Arrays.asList("D-a", "D-b", "D-c", "D-d") |
||||
); |
||||
|
||||
leftStream.join(rightStream, valueJoiner, JoinWindows.of(10000)).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testLeftKStreamKStream() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-left-KStream-KStream"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-null"), |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Arrays.asList("A-b", "B-b"), |
||||
null, |
||||
null, |
||||
Arrays.asList("C-a", "C-b"), |
||||
Arrays.asList("A-c", "B-c", "C-c"), |
||||
null, |
||||
null, |
||||
null, |
||||
Arrays.asList("A-d", "B-d", "C-d"), |
||||
Arrays.asList("D-a", "D-b", "D-c", "D-d") |
||||
); |
||||
|
||||
leftStream.leftJoin(rightStream, valueJoiner, JoinWindows.of(10000)).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testOuterKStreamKStream() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-outer-KStream-KStream"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-null"), |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Arrays.asList("A-b", "B-b"), |
||||
null, |
||||
null, |
||||
Arrays.asList("C-a", "C-b"), |
||||
Arrays.asList("A-c", "B-c", "C-c"), |
||||
null, |
||||
null, |
||||
null, |
||||
Arrays.asList("A-d", "B-d", "C-d"), |
||||
Arrays.asList("D-a", "D-b", "D-c", "D-d") |
||||
); |
||||
|
||||
leftStream.outerJoin(rightStream, valueJoiner, JoinWindows.of(10000)).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testInnerKStreamKTable() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-inner-KStream-KTable"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("B-a"), |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("D-d") |
||||
); |
||||
|
||||
leftStream.join(rightTable, valueJoiner).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testLeftKStreamKTable() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-left-KStream-KTable"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-null"), |
||||
null, |
||||
Collections.singletonList("B-a"), |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("C-null"), |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("D-d") |
||||
); |
||||
|
||||
leftStream.leftJoin(rightTable, valueJoiner).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testInnerKTableKTable() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-inner-KTable-KTable"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Collections.singletonList("B-b"), |
||||
Collections.singletonList((String) null), |
||||
null, |
||||
null, |
||||
Collections.singletonList("C-c"), |
||||
Collections.singletonList((String) null), |
||||
null, |
||||
null, |
||||
null, |
||||
Collections.singletonList("D-d") |
||||
); |
||||
|
||||
leftTable.join(rightTable, valueJoiner).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testLeftKTableKTable() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-left-KTable-KTable"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-null"), |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Collections.singletonList("B-b"), |
||||
Collections.singletonList((String) null), |
||||
null, |
||||
Collections.singletonList("C-null"), |
||||
Collections.singletonList("C-c"), |
||||
Collections.singletonList("C-null"), |
||||
Collections.singletonList((String) null), |
||||
null, |
||||
null, |
||||
Collections.singletonList("D-d") |
||||
); |
||||
|
||||
leftTable.leftJoin(rightTable, valueJoiner).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
@Test |
||||
public void testOuterKTableKTable() throws Exception { |
||||
STREAMS_CONFIG.put(StreamsConfig.APPLICATION_ID_CONFIG, APP_ID + "-outer-KTable-KTable"); |
||||
|
||||
final List<List<String>> expectedResult = Arrays.asList( |
||||
null, |
||||
null, |
||||
Collections.singletonList("A-null"), |
||||
Collections.singletonList("A-a"), |
||||
Collections.singletonList("B-a"), |
||||
Collections.singletonList("B-b"), |
||||
Collections.singletonList("null-b"), |
||||
Collections.singletonList((String) null), |
||||
Collections.singletonList("C-null"), |
||||
Collections.singletonList("C-c"), |
||||
Collections.singletonList("C-null"), |
||||
Collections.singletonList((String) null), |
||||
null, |
||||
Collections.singletonList("null-d"), |
||||
Collections.singletonList("D-d") |
||||
); |
||||
|
||||
leftTable.outerJoin(rightTable, valueJoiner).to(OUTPUT_TOPIC); |
||||
|
||||
runTest(expectedResult); |
||||
} |
||||
|
||||
private final class TopicsGotDeletedCondition implements TestCondition { |
||||
@Override |
||||
public boolean conditionMet() { |
||||
final Set<String> allTopics = new HashSet<>(); |
||||
allTopics.addAll(scala.collection.JavaConversions.seqAsJavaList(zkUtils.getAllTopics())); |
||||
return !allTopics.contains(INPUT_TOPIC_1) && !allTopics.contains(INPUT_TOPIC_2) && !allTopics.contains(OUTPUT_TOPIC); |
||||
} |
||||
} |
||||
|
||||
private final class Input<V> { |
||||
String topic; |
||||
KeyValue<Long, V> record; |
||||
|
||||
private final long anyUniqueKey = 0L; |
||||
|
||||
Input(final String topic, final V value) { |
||||
this.topic = topic; |
||||
record = KeyValue.pair(anyUniqueKey, value); |
||||
} |
||||
} |
||||
} |
@ -0,0 +1,146 @@
@@ -0,0 +1,146 @@
|
||||
/** |
||||
* 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 |
||||
* <p> |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* <p> |
||||
* 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.kafka.streams.kstream.internals; |
||||
|
||||
import org.apache.kafka.common.serialization.Serde; |
||||
import org.apache.kafka.common.serialization.Serdes; |
||||
import org.apache.kafka.streams.kstream.KStream; |
||||
import org.apache.kafka.streams.kstream.KStreamBuilder; |
||||
import org.apache.kafka.streams.kstream.KTable; |
||||
import org.apache.kafka.test.KStreamTestDriver; |
||||
import org.apache.kafka.test.MockProcessorSupplier; |
||||
import org.apache.kafka.test.MockValueJoiner; |
||||
import org.apache.kafka.test.TestUtils; |
||||
import org.junit.After; |
||||
import org.junit.Before; |
||||
import org.junit.Test; |
||||
|
||||
import java.io.File; |
||||
import java.io.IOException; |
||||
import java.util.Arrays; |
||||
import java.util.Collection; |
||||
import java.util.HashSet; |
||||
import java.util.Set; |
||||
|
||||
import static org.junit.Assert.assertEquals; |
||||
|
||||
public class KStreamKTableJoinTest { |
||||
|
||||
final private String topic1 = "topic1"; |
||||
final private String topic2 = "topic2"; |
||||
|
||||
final private Serde<Integer> intSerde = Serdes.Integer(); |
||||
final private Serde<String> stringSerde = Serdes.String(); |
||||
|
||||
private KStreamTestDriver driver = null; |
||||
private File stateDir = null; |
||||
|
||||
@After |
||||
public void tearDown() { |
||||
if (driver != null) { |
||||
driver.close(); |
||||
} |
||||
driver = null; |
||||
} |
||||
|
||||
@Before |
||||
public void setUp() throws IOException { |
||||
stateDir = TestUtils.tempDirectory("kafka-test"); |
||||
} |
||||
|
||||
@Test |
||||
public void testJoin() throws Exception { |
||||
final KStreamBuilder builder = new KStreamBuilder(); |
||||
|
||||
final int[] expectedKeys = new int[]{0, 1, 2, 3}; |
||||
|
||||
final KStream<Integer, String> stream; |
||||
final KTable<Integer, String> table; |
||||
final MockProcessorSupplier<Integer, String> processor; |
||||
|
||||
processor = new MockProcessorSupplier<>(); |
||||
stream = builder.stream(intSerde, stringSerde, topic1); |
||||
table = builder.table(intSerde, stringSerde, topic2, "anyStoreName"); |
||||
stream.join(table, MockValueJoiner.STRING_JOINER).process(processor); |
||||
|
||||
final Collection<Set<String>> copartitionGroups = builder.copartitionGroups(); |
||||
|
||||
assertEquals(1, copartitionGroups.size()); |
||||
assertEquals(new HashSet<>(Arrays.asList(topic1, topic2)), copartitionGroups.iterator().next()); |
||||
|
||||
driver = new KStreamTestDriver(builder, stateDir); |
||||
driver.setTime(0L); |
||||
|
||||
// push two items to the primary stream. the other table is empty
|
||||
|
||||
for (int i = 0; i < 2; i++) { |
||||
driver.process(topic1, expectedKeys[i], "X" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult(); |
||||
|
||||
// push two items to the other stream. this should not produce any item.
|
||||
|
||||
for (int i = 0; i < 2; i++) { |
||||
driver.process(topic2, expectedKeys[i], "Y" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult(); |
||||
|
||||
// push all four items to the primary stream. this should produce two items.
|
||||
|
||||
for (int i = 0; i < expectedKeys.length; i++) { |
||||
driver.process(topic1, expectedKeys[i], "X" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult("0:X0+Y0", "1:X1+Y1"); |
||||
|
||||
// push all items to the other stream. this should not produce any item
|
||||
for (int i = 0; i < expectedKeys.length; i++) { |
||||
driver.process(topic2, expectedKeys[i], "YY" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult(); |
||||
|
||||
// push all four items to the primary stream. this should produce four items.
|
||||
|
||||
for (int i = 0; i < expectedKeys.length; i++) { |
||||
driver.process(topic1, expectedKeys[i], "X" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult("0:X0+YY0", "1:X1+YY1", "2:X2+YY2", "3:X3+YY3"); |
||||
|
||||
// push two items with null to the other stream as deletes. this should not produce any item.
|
||||
|
||||
for (int i = 0; i < 2; i++) { |
||||
driver.process(topic2, expectedKeys[i], null); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult(); |
||||
|
||||
// push all four items to the primary stream. this should produce two items.
|
||||
|
||||
for (int i = 0; i < expectedKeys.length; i++) { |
||||
driver.process(topic1, expectedKeys[i], "XX" + expectedKeys[i]); |
||||
} |
||||
|
||||
processor.checkAndClearProcessResult("2:XX2+YY2", "3:XX3+YY3"); |
||||
} |
||||
|
||||
|
||||
} |
Loading…
Reference in new issue