Skip to content

GH-3660: Fix EmbeddedKafkaBroker seekToEnd #3661

New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Merged
merged 1 commit into from
Dec 12, 2024
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -71,6 +71,7 @@
* @author Adrian Chlebosz
* @author Soby Chacko
* @author Sanghyeok An
* @author Wouter Coekaerts
*
* @since 3.1
*/
Expand Down Expand Up @@ -560,6 +561,8 @@ public void onPartitionsAssigned(Collection<TopicPartition> partitions) {
+ (seekToEnd ? "end; " : "beginning"));
if (seekToEnd) {
consumer.seekToEnd(assigned.get());
// seekToEnd is asynchronous. query the position to force the seek to happen now.
assigned.get().forEach(consumer::position);
}
else {
consumer.seekToBeginning(assigned.get());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -89,6 +89,7 @@
* @author Soby Chacko
* @author Sanghyeok An
* @author Borahm Lee
* @author Wouter Coekaerts
*
* @since 2.2
*/
Expand Down Expand Up @@ -763,6 +764,8 @@ public void onPartitionsAssigned(Collection<TopicPartition> partitions) {
+ (seekToEnd ? "end; " : "beginning"));
if (seekToEnd) {
consumer.seekToEnd(assigned.get());
// seekToEnd is asynchronous. query the position to force the seek to happen now.
assigned.get().forEach(consumer::position);
}
else {
consumer.seekToBeginning(assigned.get());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,14 +16,21 @@

package org.springframework.kafka.test;

import java.util.Map;

import org.apache.kafka.clients.consumer.KafkaConsumer;
import org.apache.kafka.clients.producer.KafkaProducer;
import org.apache.kafka.clients.producer.ProducerRecord;
import org.junit.jupiter.api.Test;

import org.springframework.kafka.test.utils.KafkaTestUtils;
import org.springframework.util.StringUtils;

import static org.assertj.core.api.Assertions.assertThat;

/**
* @author Gary Russell
* @author Wouter Coekaerts
* @since 3.1
*
*/
Expand All @@ -37,4 +44,21 @@ void testUpDown() {
kafka.destroy();
}

@Test
void testConsumeFromEmbeddedWithSeekToEnd() {
EmbeddedKafkaKraftBroker kafka = new EmbeddedKafkaKraftBroker(1, 1, "seekTestTopic");
kafka.afterPropertiesSet();
Map<String, Object> producerProps = KafkaTestUtils.producerProps(kafka);
KafkaProducer<Integer, String> producer = new KafkaProducer<>(producerProps);
producer.send(new ProducerRecord<>("seekTestTopic", 0, 1, "beforeSeekToEnd"));
Map<String, Object> consumerProps = KafkaTestUtils.consumerProps("seekTest", "false", kafka);
KafkaConsumer<Integer, String> consumer = new KafkaConsumer<>(consumerProps);
kafka.consumeFromAnEmbeddedTopic(consumer, true /* seekToEnd */, "seekTestTopic");
producer.send(new ProducerRecord<>("seekTestTopic", 0, 1, "afterSeekToEnd"));
producer.close();
assertThat(KafkaTestUtils.getSingleRecord(consumer, "seekTestTopic").value())
.isEqualTo("afterSeekToEnd");
consumer.close();
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -16,12 +16,20 @@

package org.springframework.kafka.test;

import java.util.Map;

import org.apache.kafka.clients.consumer.KafkaConsumer;
import org.apache.kafka.clients.producer.KafkaProducer;
import org.apache.kafka.clients.producer.ProducerRecord;
import org.junit.jupiter.api.Test;

import org.springframework.kafka.test.utils.KafkaTestUtils;

import static org.assertj.core.api.Assertions.assertThat;

/**
* @author Gary Russell
* @author Wouter Coekaerts
* @since 2.3
*
*/
Expand All @@ -42,4 +50,22 @@ void testUpDown() {
assertThat(System.getProperty(EmbeddedKafkaBroker.SPRING_EMBEDDED_KAFKA_BROKERS)).isNull();
}

@Test
void testConsumeFromEmbeddedWithSeekToEnd() {
EmbeddedKafkaZKBroker kafka = new EmbeddedKafkaZKBroker(1);
kafka.afterPropertiesSet();
kafka.addTopics("seekTestTopic");
Map<String, Object> producerProps = KafkaTestUtils.producerProps(kafka);
KafkaProducer<Integer, String> producer = new KafkaProducer<>(producerProps);
producer.send(new ProducerRecord<>("seekTestTopic", 0, 1, "beforeSeekToEnd"));
Map<String, Object> consumerProps = KafkaTestUtils.consumerProps("seekTest", "false", kafka);
KafkaConsumer<Integer, String> consumer = new KafkaConsumer<>(consumerProps);
kafka.consumeFromAnEmbeddedTopic(consumer, true /* seekToEnd */, "seekTestTopic");
producer.send(new ProducerRecord<>("seekTestTopic", 0, 1, "afterSeekToEnd"));
producer.close();
assertThat(KafkaTestUtils.getSingleRecord(consumer, "seekTestTopic").value())
.isEqualTo("afterSeekToEnd");
consumer.close();
}

}
Loading