Skip to content

Commit

Permalink
Add restart for SubscribableKafkaMessageSource using the new poll met…
Browse files Browse the repository at this point in the history
…hod and add tests for the exception through the buffer for the StreamableKafkaMessageSource.
  • Loading branch information
Gerard Klijs authored and gklijs committed May 12, 2022
1 parent e625678 commit e28ef2b
Show file tree
Hide file tree
Showing 4 changed files with 171 additions and 25 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -321,5 +321,15 @@ public String toString() {
@Override
public void setException(RuntimeException exception) {
possibleException.set(exception);
try {
lock.lockInterruptibly();
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
}
try {
this.notEmpty.signal();
} finally {
lock.unlock();
}
}
}
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
/*
* Copyright (c) 2010-2021. Axon Framework
* Copyright (c) 2010-2022. Axon Framework
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
Expand Down Expand Up @@ -165,22 +165,32 @@ public void start() {
}

for (int consumerIndex = 0; consumerIndex < consumerCount; consumerIndex++) {
Consumer<K, V> consumer = consumerFactory.createConsumer(groupId);
consumer.subscribe(topics);

Registration closeConsumer = fetcher.poll(
consumer,
consumerRecords -> StreamSupport.stream(consumerRecords.spliterator(), false)
.map(messageConverter::readKafkaMessage)
.filter(Optional::isPresent)
.map(Optional::get)
.collect(Collectors.toList()),
eventMessages -> eventProcessors.forEach(eventProcessor -> eventProcessor.accept(eventMessages))
);
fetcherRegistrations.add(closeConsumer);
addConsumer();
}
}

private void addConsumer() {
Consumer<K, V> consumer = consumerFactory.createConsumer(groupId);
consumer.subscribe(topics);

Registration closeConsumer = fetcher.poll(
consumer,
consumerRecords -> StreamSupport.stream(consumerRecords.spliterator(), false)
.map(messageConverter::readKafkaMessage)
.filter(Optional::isPresent)
.map(Optional::get)
.collect(Collectors.toList()),
eventMessages -> eventProcessors.forEach(eventProcessor -> eventProcessor.accept(eventMessages)),
this::restartOnError
);
fetcherRegistrations.add(closeConsumer);
}

private void restartOnError(RuntimeException e) {
logger.warn("Consumer had a fatal exception, starting a new one", e);
addConsumer();
}

/**
* Close off this {@link SubscribableMessageSource} ensuring all used {@link Consumer}s are closed.
*/
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,126 @@
/*
* Copyright (c) 2010-2022. Axon Framework
*
* Licensed 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.axonframework.extensions.kafka.eventhandling.consumer.streamable;

import org.apache.kafka.clients.consumer.Consumer;
import org.apache.kafka.common.KafkaException;
import org.axonframework.common.stream.BlockingStream;
import org.axonframework.eventhandling.TrackedEventMessage;
import org.axonframework.extensions.kafka.eventhandling.consumer.AsyncFetcher;
import org.axonframework.extensions.kafka.eventhandling.consumer.ConsumerFactory;
import org.axonframework.extensions.kafka.eventhandling.consumer.FetchEventException;
import org.axonframework.extensions.kafka.eventhandling.consumer.Fetcher;
import org.junit.jupiter.api.*;

import java.time.Duration;
import java.util.Collections;

import static org.junit.jupiter.api.Assertions.*;
import static org.mockito.Mockito.*;
import static org.testcontainers.shaded.org.awaitility.Awaitility.await;

/**
* Kafka tests asserting an error should pop up if there is on error with the consumer. This error pops up via a call to
* {@link Buffer#setException(RuntimeException)} setException} method in the buffer which pops up once the buffer is
* empty to the consumer of the {@link StreamableKafkaMessageSource}.
*
* @author Gerard Klijs
*/
class FailingConsumerErrorThroughBufferTest {

private static final String TEST_TOPIC = "failing_consumer_test";
private Fetcher<String, byte[], KafkaEventMessage> fetcher;
private ConsumerFactory<String, byte[]> consumerFactory;
private Consumer<String, byte[]> consumer;

@BeforeEach
void setUp() {
consumer = mock(Consumer.class);
doThrow(new KafkaException("poll error"))
.when(consumer)
.poll(any());
consumerFactory = new MockFactory();
fetcher = AsyncFetcher.<String, byte[], KafkaEventMessage>builder()
.pollTimeout(300)
.build();
}

@AfterEach
void shutdown() {
fetcher.shutdown();
}

@Test
void testFetchEventExceptionAfterFailedPollWhenCallingNextAvailable() {
StreamableKafkaMessageSource<String, byte[]> streamableMessageSource =
StreamableKafkaMessageSource.<String, byte[]>builder()
.topics(Collections.singletonList(TEST_TOPIC))
.consumerFactory(consumerFactory)
.fetcher(fetcher)
.build();
BlockingStream<TrackedEventMessage<?>> stream = streamableMessageSource.openStream(null);
assertThrows(FetchEventException.class, stream::nextAvailable);
stream.close();
}

@Test
void testFetchEventExceptionAfterFailedPollWhenCallingHasNextAvailable() {
StreamableKafkaMessageSource<String, byte[]> streamableMessageSource =
StreamableKafkaMessageSource.<String, byte[]>builder()
.topics(Collections.singletonList(TEST_TOPIC))
.consumerFactory(consumerFactory)
.fetcher(fetcher)
.build();
BlockingStream<TrackedEventMessage<?>> stream = streamableMessageSource.openStream(null);
await().atMost(Duration.ofSeconds(1L)).until(() -> {
try {
stream.hasNextAvailable();
return false;
} catch (FetchEventException e) {
return true;
}
});
stream.close();
}

@Test
void testFetchEventExceptionAfterFailedPollWhenCallingPeek() {
StreamableKafkaMessageSource<String, byte[]> streamableMessageSource =
StreamableKafkaMessageSource.<String, byte[]>builder()
.topics(Collections.singletonList(TEST_TOPIC))
.consumerFactory(consumerFactory)
.fetcher(fetcher)
.build();
BlockingStream<TrackedEventMessage<?>> stream = streamableMessageSource.openStream(null);
await().atMost(Duration.ofSeconds(1L)).until(() -> {
try {
stream.peek();
return false;
} catch (FetchEventException e) {
return true;
}
});
stream.close();
}

private class MockFactory implements ConsumerFactory<String, byte[]> {

@Override
public Consumer<String, byte[]> createConsumer(String groupId) {
return consumer;
}
}
}
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
/*
* Copyright (c) 2010-2021. Axon Framework
* Copyright (c) 2010-2022. Axon Framework
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
Expand Down Expand Up @@ -127,7 +127,7 @@ void testBuildingWhilstMissingRequiredFieldsShouldThrowAxonConfigurationExceptio

@Test
void testAutoStartInitiatesProcessingOnFirstEventProcessor() {
when(fetcher.poll(eq(mockConsumer), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);
when(fetcher.poll(eq(mockConsumer), any(), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);

SubscribableKafkaMessageSource<String, String> testSubject =
SubscribableKafkaMessageSource.<String, String>builder()
Expand All @@ -142,13 +142,13 @@ void testAutoStartInitiatesProcessingOnFirstEventProcessor() {

verify(consumerFactory, times(1)).createConsumer(DEFAULT_GROUP_ID);
verify(mockConsumer, times(1)).subscribe(Collections.singletonList(TEST_TOPIC));
verify(fetcher).poll(eq(mockConsumer), any(), any());
verify(fetcher).poll(eq(mockConsumer), any(), any(), any());
}

@Test
void testCancelingSubscribedEventProcessorRunsConnectedCloseHandlerWhenAutoStartIsOn() {
AtomicBoolean closedFetcherRegistration = new AtomicBoolean(false);
when(fetcher.poll(eq(mockConsumer), any(), any())).thenReturn(() -> {
when(fetcher.poll(eq(mockConsumer), any(), any(), any())).thenReturn(() -> {
closedFetcherRegistration.set(true);
return true;
});
Expand All @@ -174,7 +174,7 @@ void testCancelingSubscribedEventProcessorRunsConnectedCloseHandlerWhenAutoStart

@Test
void testSubscribingTheSameInstanceTwiceDisregardsSecondInstanceOnStart() {
when(fetcher.poll(eq(mockConsumer), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);
when(fetcher.poll(eq(mockConsumer), any(), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);

testSubject.subscribe(NO_OP_EVENT_PROCESSOR);
testSubject.subscribe(NO_OP_EVENT_PROCESSOR);
Expand All @@ -183,7 +183,7 @@ void testSubscribingTheSameInstanceTwiceDisregardsSecondInstanceOnStart() {

verify(consumerFactory, times(1)).createConsumer(DEFAULT_GROUP_ID);
verify(mockConsumer, times(1)).subscribe(Collections.singletonList(TEST_TOPIC));
verify(fetcher, times(1)).poll(eq(mockConsumer), any(), any());
verify(fetcher, times(1)).poll(eq(mockConsumer), any(), any(), any());
}

@Test
Expand All @@ -207,14 +207,14 @@ void testStartSubscribesConsumerToAllProvidedTopics() {

verify(consumerFactory).createConsumer(DEFAULT_GROUP_ID);
verify(mockConsumer).subscribe(testTopics);
verify(fetcher).poll(eq(mockConsumer), any(), any());
verify(fetcher).poll(eq(mockConsumer), any(), any(), any());
}

@Test
void testStartBuildsConsumersUpToConsumerCount() {
int expectedNumberOfConsumers = 2;

when(fetcher.poll(eq(mockConsumer), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);
when(fetcher.poll(eq(mockConsumer), any(), any(), any())).thenReturn(NO_OP_FETCHER_REGISTRATION);

SubscribableKafkaMessageSource<String, String> testSubject =
SubscribableKafkaMessageSource.<String, String>builder()
Expand All @@ -230,7 +230,7 @@ void testStartBuildsConsumersUpToConsumerCount() {

verify(consumerFactory, times(expectedNumberOfConsumers)).createConsumer(DEFAULT_GROUP_ID);
verify(mockConsumer, times(expectedNumberOfConsumers)).subscribe(Collections.singletonList(TEST_TOPIC));
verify(fetcher, times(expectedNumberOfConsumers)).poll(eq(mockConsumer), any(), any());
verify(fetcher, times(expectedNumberOfConsumers)).poll(eq(mockConsumer), any(), any(), any());
}

@Test
Expand All @@ -239,7 +239,7 @@ void testCloseRunsCloseHandlerPerConsumerCount() {

AtomicBoolean closedEventProcessorOne = new AtomicBoolean(false);
AtomicBoolean closedEventProcessorTwo = new AtomicBoolean(false);
when(fetcher.poll(eq(mockConsumer), any(), any()))
when(fetcher.poll(eq(mockConsumer), any(), any(), any()))
.thenReturn(() -> {
closedEventProcessorOne.set(true);
return true;
Expand All @@ -264,7 +264,7 @@ void testCloseRunsCloseHandlerPerConsumerCount() {

verify(consumerFactory, times(expectedNumberOfConsumers)).createConsumer(DEFAULT_GROUP_ID);
verify(mockConsumer, times(expectedNumberOfConsumers)).subscribe(Collections.singletonList(TEST_TOPIC));
verify(fetcher, times(expectedNumberOfConsumers)).poll(eq(mockConsumer), any(), any());
verify(fetcher, times(expectedNumberOfConsumers)).poll(eq(mockConsumer), any(), any(), any());

assertTrue(closedEventProcessorOne.get());
assertTrue(closedEventProcessorTwo.get());
Expand Down

0 comments on commit e28ef2b

Please sign in to comment.