3pacccccc opened a new pull request, #24663: URL: https://github.com/apache/pulsar/pull/24663
Fixes https://github.com/apache/pulsar/issues/24541 ### Motivation When a consumer with a dead-letter policy (DLQ) encounters a message with a schema mismatch (e.g., producer sends Long but consumer expects String), and the message exceeds the maximum redelivery count, the following infinite loop occurs: 1. The consumer tries to send the message to the DLQ. 2. The DLQ send fails with a SchemaSerializationException due to the schema mismatch. 3. The consumer returns false from processPossibleToDLQ, causing the message to be redelivered. 4. The message is redelivered, exceeds the redelivery count again, and the process repeats indefinitely. here's the example how to reproduce this issue: ```java public void sendDeadLetterTopicWithMismatchSchemaProducer() throws Exception { String namespace = BrokerTestUtil.newUniqueName("my-property/my-ns"); admin.namespaces().createNamespace(namespace); // don't enforce schema validation admin.namespaces().setSchemaValidationEnforced(namespace, false); // set schema compatibility strategy to always compatible admin.namespaces().setSchemaCompatibilityStrategy(namespace, SchemaCompatibilityStrategy.ALWAYS_COMPATIBLE); String topic = BrokerTestUtil.newUniqueName("persistent://" + namespace + "/sendDeadLetterTopicWithMismatchSchemaProducer"); // create topics admin.topics().createNonPartitionedTopic(topic); admin.topics().createNonPartitionedTopic(topic + "-DLQ"); admin.topics().createNonPartitionedTopic(topic + "-RETRY"); final int maxRedeliverCount = 1; final String subscriptionName = "my-subscription"; Consumer<String> consumer = pulsarClient.newConsumer(Schema.AVRO(String.class)) .topic(topic) .subscriptionType(SubscriptionType.Shared) .subscriptionName(subscriptionName) .deadLetterPolicy(DeadLetterPolicy.builder() .deadLetterTopic(topic + "-DLQ") .retryLetterTopic(topic + "-RETRY") .maxRedeliverCount(maxRedeliverCount) .build()) .negativeAckRedeliveryDelay(1, TimeUnit.SECONDS) .messageListener((Consumer::negativeAcknowledge)) .subscribe(); Producer<Long> producer = pulsarClient.newProducer(Schema.AVRO(Long.class)).topic(topic).create(); producer.send(1234567890L); } ``` and here's what's going on behind this scenario: | producer | broker | consumer | |--------------------------------------|---------------------------------|-----------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------| | sends message with AVRO(Long) schema | | | | | delivery to consumer | | | | | received message and nack this message | | | delivery to consumer | | | | | received message and nack this message | | | note: ***infinity loop start*** | | | delivery to consumer | | | | | exceed ```maxRedeliverCount``` in ```deadLetterPolicy```, send to ```deadLetterTopic``` | | | | send to ```deadLetterTopic``` failed due to mismatch schema<sup>[1]</sup>, and result will be false in ```ConsumerImpl.processPossibleToDLQ```<sup>[2]</sup> | | | | message will redelivery to broker<sup>[3]</sup> | | | delivery to consumer | | | | | exceed ```maxRedeliverCount``` in ```deadLetterPolicy```, send to ```deadLetterTopic``` | | | | send to ```deadLetterTopic``` failed with ```SchemaSerializationException```due to mismatch schema<sup>[1]</sup>, and result will be false in ```ConsumerImpl.processPossibleToDLQ```<sup>[2]</sup> | | | | message will redelivery to broker<sup>[3]</sup> | | | delivery to consumer | | | | | exceed ```maxRedeliverCount``` in ```deadLetterPolicy```, send to ```deadLetterTopic``` | | | | send to ```deadLetterTopic``` failed due to mismatch schema<sup>[1]</sup>, and result will be false in ```ConsumerImpl.processPossibleToDLQ```<sup>[2]</sup> | | | | message will redelivery to broker<sup>[3]</sup> | | | ... | | [1]:https://github.com/apache/pulsar/blob/e606aeeebb6cc02f154758a9ae56fb85b621f8e0/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerImpl.java#L2335 [2]:https://github.com/apache/pulsar/blob/e606aeeebb6cc02f154758a9ae56fb85b621f8e0/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerImpl.java#L2379 [3]:https://github.com/apache/pulsar/blob/e606aeeebb6cc02f154758a9ae56fb85b621f8e0/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerImpl.java#L2264 ### Modifications - Catch SchemaSerializationException in processPossibleToDLQ: When this exception occurs during DLQ send, mark the message as "processed" (i.e., complete with true) to prevent redelivery and break the loop. - Improve logging: - Fix ambiguous debug log format from {}/{} to {}:{} for ledger and entry ID. - Unify and clarify error logs in processPossibleToDLQ to better distinguish between ProducerQueueIsFullError and other exceptions. ### Verifying this change - [x] Make sure that the change passes the CI checks. <!-- DO NOT REMOVE THIS SECTION. CHECK THE PROPER BOX ONLY. --> *If the box was checked, please highlight the changes* - [ ] Dependencies (add or upgrade a dependency) - [ ] The public API - [ ] The schema - [ ] The default values of configurations - [ ] The threading model - [ ] The binary protocol - [ ] The REST endpoints - [ ] The admin CLI options - [ ] The metrics - [ ] Anything that affects deployment ### Documentation <!-- DO NOT REMOVE THIS SECTION. CHECK THE PROPER BOX ONLY. --> - [ ] `doc` <!-- Your PR contains doc changes. --> - [ ] `doc-required` <!-- Your PR changes impact docs and you will update later --> - [x] `doc-not-needed` <!-- Your PR changes do not impact docs --> - [ ] `doc-complete` <!-- Docs have been already added --> ### Matching PR in forked repository PR in forked repository: https://github.com/3pacccccc/pulsar/pull/24 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: [email protected] For queries about this service, please contact Infrastructure at: [email protected]
