mirror of https://github.com/apache/kafka.git
KAFKA-19511: Fix flaky test HandlingSourceTopicDeletionIntegrationTest.shouldThrowErrorAfterSourceTopicDeleted (#20187)
CI / build (push) Waiting to run
Details
CI / build (push) Waiting to run
Details
Temporarily fix it by disable the new protocol, will take a deeper look at it in the consumer protocol. Reviewers: Matthias J. Sax <matthias@confluent.io>
This commit is contained in:
parent
9b542b6ea2
commit
38e3359446
|
@ -17,7 +17,6 @@
|
|||
package org.apache.kafka.streams.integration;
|
||||
|
||||
import org.apache.kafka.common.serialization.Serdes;
|
||||
import org.apache.kafka.streams.GroupProtocol;
|
||||
import org.apache.kafka.streams.KafkaStreams;
|
||||
import org.apache.kafka.streams.KafkaStreams.State;
|
||||
import org.apache.kafka.streams.StreamsBuilder;
|
||||
|
@ -34,12 +33,10 @@ import org.junit.jupiter.api.AfterEach;
|
|||
import org.junit.jupiter.api.BeforeAll;
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Tag;
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.api.TestInfo;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.Locale;
|
||||
import java.util.Properties;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
|
||||
|
@ -78,9 +75,8 @@ public class HandlingSourceTopicDeletionIntegrationTest {
|
|||
CLUSTER.deleteTopics(INPUT_TOPIC, OUTPUT_TOPIC);
|
||||
}
|
||||
|
||||
@ParameterizedTest
|
||||
@ValueSource(booleans = {false, true})
|
||||
public void shouldThrowErrorAfterSourceTopicDeleted(final boolean useNewProtocol, final TestInfo testName) throws InterruptedException {
|
||||
@Test
|
||||
public void shouldThrowErrorAfterSourceTopicDeleted(final TestInfo testName) throws InterruptedException {
|
||||
final StreamsBuilder builder = new StreamsBuilder();
|
||||
builder.stream(INPUT_TOPIC, Consumed.with(Serdes.Integer(), Serdes.String()))
|
||||
.to(OUTPUT_TOPIC, Produced.with(Serdes.Integer(), Serdes.String()));
|
||||
|
@ -91,25 +87,24 @@ public class HandlingSourceTopicDeletionIntegrationTest {
|
|||
final Properties streamsConfiguration = new Properties();
|
||||
streamsConfiguration.put(StreamsConfig.APPLICATION_ID_CONFIG, appId);
|
||||
streamsConfiguration.put(StreamsConfig.BOOTSTRAP_SERVERS_CONFIG, CLUSTER.bootstrapServers());
|
||||
streamsConfiguration.put(StreamsConfig.DEFAULT_KEY_SERDE_CLASS_CONFIG, Serdes.Integer().getClass());
|
||||
streamsConfiguration.put(StreamsConfig.DEFAULT_VALUE_SERDE_CLASS_CONFIG, Serdes.String().getClass());
|
||||
streamsConfiguration.put(StreamsConfig.DEFAULT_KEY_SERDE_CLASS_CONFIG, Serdes.IntegerSerde.class);
|
||||
streamsConfiguration.put(StreamsConfig.DEFAULT_VALUE_SERDE_CLASS_CONFIG, Serdes.StringSerde.class);
|
||||
streamsConfiguration.put(StreamsConfig.NUM_STREAM_THREADS_CONFIG, NUM_THREADS);
|
||||
streamsConfiguration.put(StreamsConfig.METADATA_MAX_AGE_CONFIG, 2000);
|
||||
|
||||
if (useNewProtocol) {
|
||||
streamsConfiguration.put(StreamsConfig.GROUP_PROTOCOL_CONFIG, GroupProtocol.STREAMS.name().toLowerCase(Locale.getDefault()));
|
||||
}
|
||||
|
||||
final Topology topology = builder.build();
|
||||
final KafkaStreams kafkaStreams1 = new KafkaStreams(topology, streamsConfiguration);
|
||||
final AtomicBoolean calledUncaughtExceptionHandler1 = new AtomicBoolean(false);
|
||||
final AtomicBoolean calledUncaughtExceptionHandler2 = new AtomicBoolean(false);
|
||||
|
||||
try (final KafkaStreams kafkaStreams1 = new KafkaStreams(topology, streamsConfiguration);
|
||||
final KafkaStreams kafkaStreams2 = new KafkaStreams(topology, streamsConfiguration)) {
|
||||
|
||||
kafkaStreams1.setUncaughtExceptionHandler(exception -> {
|
||||
calledUncaughtExceptionHandler1.set(true);
|
||||
return StreamsUncaughtExceptionHandler.StreamThreadExceptionResponse.SHUTDOWN_CLIENT;
|
||||
});
|
||||
kafkaStreams1.start();
|
||||
final KafkaStreams kafkaStreams2 = new KafkaStreams(topology, streamsConfiguration);
|
||||
final AtomicBoolean calledUncaughtExceptionHandler2 = new AtomicBoolean(false);
|
||||
|
||||
kafkaStreams2.setUncaughtExceptionHandler(exception -> {
|
||||
calledUncaughtExceptionHandler2.set(true);
|
||||
return StreamsUncaughtExceptionHandler.StreamThreadExceptionResponse.SHUTDOWN_CLIENT;
|
||||
|
@ -134,3 +129,4 @@ public class HandlingSourceTopicDeletionIntegrationTest {
|
|||
assertThat(calledUncaughtExceptionHandler2.get(), is(true));
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
Loading…
Reference in New Issue