|
41 | 41 | import org.apache.kafka.common.errors.InvalidConfigurationException;
|
42 | 42 | import org.apache.kafka.common.errors.InvalidRecordStateException;
|
43 | 43 | import org.apache.kafka.common.errors.InvalidTopicException;
|
| 44 | +import org.apache.kafka.common.errors.RecordDeserializationException; |
| 45 | +import org.apache.kafka.common.errors.SerializationException; |
44 | 46 | import org.apache.kafka.common.errors.WakeupException;
|
45 | 47 | import org.apache.kafka.common.header.Header;
|
46 | 48 | import org.apache.kafka.common.header.Headers;
|
|
67 | 69 | import org.junit.jupiter.api.Tag;
|
68 | 70 | import org.junit.jupiter.api.Timeout;
|
69 | 71 |
|
| 72 | +import java.nio.ByteBuffer; |
70 | 73 | import java.time.Duration;
|
71 | 74 | import java.util.ArrayList;
|
72 | 75 | import java.util.Arrays;
|
@@ -843,6 +846,144 @@ public void testExplicitAcknowledgeThrowsNotInBatch() {
|
843 | 846 | }
|
844 | 847 | }
|
845 | 848 |
|
| 849 | + @ClusterTest |
| 850 | + public void testExplicitOverrideAcknowledgeCorruptedMessage() { |
| 851 | + alterShareAutoOffsetReset("group1", "earliest"); |
| 852 | + try (Producer<byte[], byte[]> producer = createProducer(); |
| 853 | + ShareConsumer<byte[], byte[]> shareConsumer = createShareConsumer( |
| 854 | + "group1", |
| 855 | + Map.of(ConsumerConfig.SHARE_ACKNOWLEDGEMENT_MODE_CONFIG, EXPLICIT), |
| 856 | + null, |
| 857 | + mockErrorDeserializer(3))) { |
| 858 | + |
| 859 | + ProducerRecord<byte[], byte[]> record1 = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 860 | + ProducerRecord<byte[], byte[]> record2 = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 861 | + ProducerRecord<byte[], byte[]> record3 = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 862 | + producer.send(record1); |
| 863 | + producer.send(record2); |
| 864 | + producer.send(record3); |
| 865 | + producer.flush(); |
| 866 | + |
| 867 | + shareConsumer.subscribe(Set.of(tp.topic())); |
| 868 | + |
| 869 | + ConsumerRecords<byte[], byte[]> records = shareConsumer.poll(Duration.ofSeconds(60)); |
| 870 | + assertEquals(2, records.count()); |
| 871 | + Iterator<ConsumerRecord<byte[], byte[]>> iterator = records.iterator(); |
| 872 | + |
| 873 | + ConsumerRecord<byte[], byte[]> firstRecord = iterator.next(); |
| 874 | + ConsumerRecord<byte[], byte[]> secondRecord = iterator.next(); |
| 875 | + assertEquals(0L, firstRecord.offset()); |
| 876 | + assertEquals(1L, secondRecord.offset()); |
| 877 | + shareConsumer.acknowledge(firstRecord); |
| 878 | + shareConsumer.acknowledge(secondRecord); |
| 879 | + |
| 880 | + RecordDeserializationException rde = assertThrows(RecordDeserializationException.class, () -> shareConsumer.poll(Duration.ofSeconds(60))); |
| 881 | + assertEquals(2, rde.offset()); |
| 882 | + shareConsumer.commitSync(); |
| 883 | + |
| 884 | + // The corrupted record was automatically released, so we can still obtain it. |
| 885 | + rde = assertThrows(RecordDeserializationException.class, () -> shareConsumer.poll(Duration.ofSeconds(60))); |
| 886 | + assertEquals(2, rde.offset()); |
| 887 | + |
| 888 | + // Reject this record |
| 889 | + shareConsumer.acknowledge(rde.topicPartition().topic(), rde.topicPartition().partition(), rde.offset(), AcknowledgeType.REJECT); |
| 890 | + shareConsumer.commitSync(); |
| 891 | + |
| 892 | + records = shareConsumer.poll(Duration.ZERO); |
| 893 | + assertEquals(0, records.count()); |
| 894 | + verifyShareGroupStateTopicRecordsProduced(); |
| 895 | + } |
| 896 | + } |
| 897 | + |
| 898 | + @ClusterTest |
| 899 | + public void testExplicitAcknowledgeOffsetThrowsNotException() { |
| 900 | + alterShareAutoOffsetReset("group1", "earliest"); |
| 901 | + try (Producer<byte[], byte[]> producer = createProducer(); |
| 902 | + ShareConsumer<byte[], byte[]> shareConsumer = createShareConsumer( |
| 903 | + "group1", |
| 904 | + Map.of(ConsumerConfig.SHARE_ACKNOWLEDGEMENT_MODE_CONFIG, EXPLICIT))) { |
| 905 | + |
| 906 | + ProducerRecord<byte[], byte[]> record = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 907 | + producer.send(record); |
| 908 | + producer.flush(); |
| 909 | + |
| 910 | + shareConsumer.subscribe(Set.of(tp.topic())); |
| 911 | + |
| 912 | + ConsumerRecords<byte[], byte[]> records = shareConsumer.poll(Duration.ofSeconds(60)); |
| 913 | + assertEquals(1, records.count()); |
| 914 | + ConsumerRecord<byte[], byte[]> consumedRecord = records.records(tp).get(0); |
| 915 | + assertEquals(0L, consumedRecord.offset()); |
| 916 | + |
| 917 | + assertThrows(IllegalStateException.class, () -> shareConsumer.acknowledge(tp.topic(), tp.partition(), consumedRecord.offset(), AcknowledgeType.ACCEPT)); |
| 918 | + |
| 919 | + shareConsumer.acknowledge(consumedRecord); |
| 920 | + verifyShareGroupStateTopicRecordsProduced(); |
| 921 | + } |
| 922 | + } |
| 923 | + |
| 924 | + @ClusterTest |
| 925 | + public void testExplicitAcknowledgeOffsetThrowsParametersError() { |
| 926 | + alterShareAutoOffsetReset("group1", "earliest"); |
| 927 | + try (Producer<byte[], byte[]> producer = createProducer(); |
| 928 | + ShareConsumer<byte[], byte[]> shareConsumer = createShareConsumer( |
| 929 | + "group1", |
| 930 | + Map.of(ConsumerConfig.SHARE_ACKNOWLEDGEMENT_MODE_CONFIG, EXPLICIT), |
| 931 | + null, |
| 932 | + mockErrorDeserializer(2))) { |
| 933 | + |
| 934 | + ProducerRecord<byte[], byte[]> record1 = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 935 | + ProducerRecord<byte[], byte[]> record2 = new ProducerRecord<>(tp.topic(), tp.partition(), null, "key".getBytes(), "value".getBytes()); |
| 936 | + producer.send(record1); |
| 937 | + producer.send(record2); |
| 938 | + producer.flush(); |
| 939 | + |
| 940 | + shareConsumer.subscribe(Set.of(tp.topic())); |
| 941 | + |
| 942 | + ConsumerRecords<byte[], byte[]> records = shareConsumer.poll(Duration.ofSeconds(60)); |
| 943 | + assertEquals(1, records.count()); |
| 944 | + Iterator<ConsumerRecord<byte[], byte[]>> iterator = records.iterator(); |
| 945 | + |
| 946 | + ConsumerRecord<byte[], byte[]> firstRecord = iterator.next(); |
| 947 | + assertEquals(0L, firstRecord.offset()); |
| 948 | + shareConsumer.acknowledge(firstRecord); |
| 949 | + |
| 950 | + final RecordDeserializationException rde = assertThrows(RecordDeserializationException.class, () -> shareConsumer.poll(Duration.ofSeconds(60))); |
| 951 | + assertEquals(1, rde.offset()); |
| 952 | + |
| 953 | + assertThrows(IllegalStateException.class, () -> shareConsumer.acknowledge("foo", rde.topicPartition().partition(), rde.offset(), AcknowledgeType.REJECT)); |
| 954 | + assertThrows(IllegalStateException.class, () -> shareConsumer.acknowledge(rde.topicPartition().topic(), 1, rde.offset(), AcknowledgeType.REJECT)); |
| 955 | + assertThrows(IllegalStateException.class, () -> shareConsumer.acknowledge(rde.topicPartition().topic(), tp2.partition(), 0, AcknowledgeType.REJECT)); |
| 956 | + |
| 957 | + // Reject this record. |
| 958 | + shareConsumer.acknowledge(rde.topicPartition().topic(), rde.topicPartition().partition(), rde.offset(), AcknowledgeType.REJECT); |
| 959 | + shareConsumer.commitSync(); |
| 960 | + |
| 961 | + // The next acknowledge() should throw an IllegalStateException as the record has been acked. |
| 962 | + assertThrows(IllegalStateException.class, () -> shareConsumer.acknowledge(rde.topicPartition().topic(), rde.topicPartition().partition(), rde.offset(), AcknowledgeType.REJECT)); |
| 963 | + |
| 964 | + records = shareConsumer.poll(Duration.ZERO); |
| 965 | + assertEquals(0, records.count()); |
| 966 | + verifyShareGroupStateTopicRecordsProduced(); |
| 967 | + } |
| 968 | + } |
| 969 | + |
| 970 | + private ByteArrayDeserializer mockErrorDeserializer(int recordNumber) { |
| 971 | + int recordIndex = recordNumber - 1; |
| 972 | + return new ByteArrayDeserializer() { |
| 973 | + int i = 0; |
| 974 | + |
| 975 | + @Override |
| 976 | + public byte[] deserialize(String topic, Headers headers, ByteBuffer data) { |
| 977 | + if (i == recordIndex) { |
| 978 | + throw new SerializationException(); |
| 979 | + } else { |
| 980 | + i++; |
| 981 | + return super.deserialize(topic, headers, data); |
| 982 | + } |
| 983 | + } |
| 984 | + }; |
| 985 | + } |
| 986 | + |
846 | 987 | @ClusterTest
|
847 | 988 | public void testImplicitAcknowledgeFailsExplicit() {
|
848 | 989 | alterShareAutoOffsetReset("group1", "earliest");
|
@@ -2794,13 +2935,22 @@ private <K, V> ShareConsumer<K, V> createShareConsumer(String groupId) {
|
2794 | 2935 | private <K, V> ShareConsumer<K, V> createShareConsumer(
|
2795 | 2936 | String groupId,
|
2796 | 2937 | Map<?, ?> additionalProperties
|
| 2938 | + ) { |
| 2939 | + return createShareConsumer(groupId, additionalProperties, null, null); |
| 2940 | + } |
| 2941 | + |
| 2942 | + private <K, V> ShareConsumer<K, V> createShareConsumer( |
| 2943 | + String groupId, |
| 2944 | + Map<?, ?> additionalProperties, |
| 2945 | + Deserializer<K> keyDeserializer, |
| 2946 | + Deserializer<V> valueDeserializer |
2797 | 2947 | ) {
|
2798 | 2948 | Properties props = new Properties();
|
2799 | 2949 | props.putAll(additionalProperties);
|
2800 | 2950 | props.put(ConsumerConfig.GROUP_ID_CONFIG, groupId);
|
2801 | 2951 | Map<String, Object> conf = new HashMap<>();
|
2802 | 2952 | props.forEach((k, v) -> conf.put((String) k, v));
|
2803 |
| - return cluster.shareConsumer(conf); |
| 2953 | + return cluster.shareConsumer(conf, keyDeserializer, valueDeserializer); |
2804 | 2954 | }
|
2805 | 2955 |
|
2806 | 2956 | private void warmup() throws InterruptedException {
|
|
0 commit comments