You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pulsar.apache.org by GitBox <gi...@apache.org> on 2022/02/16 10:50:47 UTC

[GitHub] [pulsar] Shawyeok opened a new pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Shawyeok opened a new pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317


   <!--
   ### Contribution Checklist
     
     - Name the pull request in the form "[Issue XYZ][component] Title of the pull request", where *XYZ* should be replaced by the actual issue number.
       Skip *Issue XYZ* if there is no associated github issue for this pull request.
       Skip *component* if you are unsure about which is the best component. E.g. `[docs] Fix typo in produce method`.
   
     - Fill out the template below to describe the changes contributed by the pull request. That will give reviewers the context they need to do the review.
     
     - Each pull request should address only one issue, not mix up code from multiple issues.
     
     - Each commit in the pull request has a meaningful commit message
   
     - Once all items of the checklist are addressed, remove the above text and this checklist, leaving only the filled out template below.
   
   **(The sections below can be removed for hotfixes of typos)**
   -->
   
   ### Motivation
   
   If a message reached maxRedeliverCount, it will send to deadLetterTopic, since `2.8.0`, this mechanism is **broken**, it was introduced in #9970
   
   You can reproduce with code below:
   ```java
   class Scratch {
   
       private static final Logger LOG = LoggerFactory.getLogger(Scratch.class);
   
       public static void main(String[] args) throws Exception {
           PulsarClient pulsarClient = PulsarClient.builder()
                   .serviceUrl(args[0])
                   .build();
           DeadLetterPolicy deadLetterPolicy = DeadLetterPolicy.builder()
                   .maxRedeliverCount(0)
                   .build();
           Consumer<byte[]> consumer = pulsarClient.newConsumer()
                   .topic(args[1])
                   .subscriptionName("consumeTest")
                   .subscriptionType(SubscriptionType.Shared)
                   .deadLetterPolicy(deadLetterPolicy)
                   .ackTimeout(10, TimeUnit.SECONDS)
                   .subscribe();
           int i = 1;
           Message<byte[]> message;
           while ((message = consumer.receive()) != null) {
               MessageId messageId = message.getMessageId();
               LOG.info("Receive a message {}: {}", messageId, new String(message.getData()));
               if (i-- > 0) {
                   LOG.info("Skip a message {}", messageId);
                   continue;
               }
               consumer.acknowledge(messageId);
           }
           new CountDownLatch(1).await();
       }
   }
   ```
   
   It will log exception below:
   ```
   Dead letter producer exception with topic: {{topic}}
   java.util.concurrent.CompletionException: java.lang.NullPointerException
   	at java.util.concurrent.CompletableFuture.encodeThrowable(CompletableFuture.java:273)
   	at java.util.concurrent.CompletableFuture.completeThrowable(CompletableFuture.java:280)
   	at java.util.concurrent.CompletableFuture.uniAccept(CompletableFuture.java:659)
   	at java.util.concurrent.CompletableFuture$UniAccept.tryFire(CompletableFuture.java:632)
   	at java.util.concurrent.CompletableFuture$Completion.exec(CompletableFuture.java:443)
   	at java.util.concurrent.ForkJoinTask.doExec(ForkJoinTask.java:289)
   	at java.util.concurrent.ForkJoinPool$WorkQueue.runTask(ForkJoinPool.java:1056)
   	at java.util.concurrent.ForkJoinPool.runWorker(ForkJoinPool.java:1692)
   	at java.util.concurrent.ForkJoinWorkerThread.run(ForkJoinWorkerThread.java:157)
   Caused by: java.lang.NullPointerException
   	at java.util.Objects.requireNonNull(Objects.java:203)
   	at org.apache.pulsar.client.impl.schema.AbstractSchema.atSchemaVersion(AbstractSchema.java:81)
   	at org.apache.pulsar.client.impl.MessageImpl.getReaderSchema(MessageImpl.java:398)
   	at org.apache.pulsar.client.impl.ConsumerImpl.lambda$processPossibleToDLQ$38(ConsumerImpl.java:1692)
   	at java.util.concurrent.CompletableFuture.uniAccept(CompletableFuture.java:656)
   	... 6 more
   ```
   
   ```java
       // MessageImpl#getReaderSchema
       public Optional<Schema<?>> getReaderSchema() {
           ensureSchemaIsLoaded();
           if (schema == null) {
               return Optional.empty();
           }
           if (schema instanceof AutoConsumeSchema) {
               byte[] schemaVersion = getSchemaVersion();
               return Optional.of(((AutoConsumeSchema) schema)
                       .atSchemaVersion(schemaVersion));
           } else if (schema instanceof AbstractSchema) {
               byte[] schemaVersion = getSchemaVersion();    // schemaVersion may be null,  e.g. BYTES schema
               return Optional.of(((AbstractSchema<?>) schema)
                       .atSchemaVersion(schemaVersion));           // if schemaVersion is null, a NPE will throw
           } else {
               return Optional.of(schema);
           }
       }
   ```
   
   ### Modifications
   
   Make `AbstractSchema#atSchemaVersion` throw NPE only if `supportSchemaVersioning` is true and schemaVersion is null.
   
   ### Verifying this change
   
   This change added tests and can be verified as follows:
   
     - *Added a unit test for MessageImpl#getReaderSchema with a message which has BYTES schema*
   
   ### Does this pull request potentially affect one of the following parts:
   
   *If `yes` was chosen, please highlight the changes*
   
     - Dependencies (does it add or upgrade a dependency): (no)
     - The public API: (no)
     - The schema: (no)
     - The default values of configurations: (no)
     - The wire protocol: (no)
     - The rest endpoints: (no)
     - The admin cli options: (no)
     - Anything that affects deployment: (no)
   
   ### Documentation
   
   Check the box below or label this PR directly (if you have committer privilege).
   
   Need to update docs? 
   
   - [ ] `doc-required` 
     
     (If you need help on updating docs, create a doc issue)
     
   - [x] `no-need-doc` 
     
   Bugfix only
     
   - [ ] `doc` 
     
     (If this PR contains doc changes)
   
   
   


-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] Technoboy- commented on a change in pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
Technoboy- commented on a change in pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317#discussion_r808883245



##########
File path: pulsar-client/src/main/java/org/apache/pulsar/client/impl/schema/AbstractSchema.java
##########
@@ -74,14 +74,13 @@ public T decode(ByteBuf byteBuf, byte[] schemaVersion) {
      * @param schemaVersion the version
      * @return the schema at that specific version
      * @throws SchemaSerializationException in case of unknown schema version
-     * @throws NullPointerException in case of null schemaVersion
+     * @throws NullPointerException in case of null schemaVersion and supportSchemaVersioning is true
      */
     public Schema<?> atSchemaVersion(byte[] schemaVersion) throws SchemaSerializationException {
-        Objects.requireNonNull(schemaVersion);
         if (!supportSchemaVersioning()) {
             return this;
-        } else {
-            throw new SchemaSerializationException("Not implemented for " + this.getClass());
         }
+        Objects.requireNonNull(schemaVersion);

Review comment:
       Why do we need to check null for `schemaVersion` ?  It's very strange here and outside return Optional.
   ```
   else if (schema instanceof AbstractSchema) {
               byte[] schemaVersion = getSchemaVersion();
               return Optional.of(((AbstractSchema<?>) schema)
                       .atSchemaVersion(schemaVersion));          
           }
   ```




-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] 315157973 merged pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
315157973 merged pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317


   


-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] Technoboy- commented on a change in pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
Technoboy- commented on a change in pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317#discussion_r808883480



##########
File path: pulsar-client/src/main/java/org/apache/pulsar/client/impl/schema/AbstractSchema.java
##########
@@ -74,14 +74,13 @@ public T decode(ByteBuf byteBuf, byte[] schemaVersion) {
      * @param schemaVersion the version
      * @return the schema at that specific version
      * @throws SchemaSerializationException in case of unknown schema version
-     * @throws NullPointerException in case of null schemaVersion
+     * @throws NullPointerException in case of null schemaVersion and supportSchemaVersioning is true
      */
     public Schema<?> atSchemaVersion(byte[] schemaVersion) throws SchemaSerializationException {
-        Objects.requireNonNull(schemaVersion);
         if (!supportSchemaVersioning()) {
             return this;
-        } else {
-            throw new SchemaSerializationException("Not implemented for " + this.getClass());
         }
+        Objects.requireNonNull(schemaVersion);

Review comment:
       @congbobo184 




-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] michaeljmarshall commented on a change in pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
michaeljmarshall commented on a change in pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317#discussion_r813587681



##########
File path: pulsar-client/src/test/java/org/apache/pulsar/client/impl/MessageTest.java
##########
@@ -81,4 +81,15 @@ public void testTopicMessageImplNoReplicatedInfo() {
         assertFalse(topicMessage.isReplicated());
         assertNull(topicMessage.getReplicatedFrom());
     }
+
+    @Test
+    public void testMessageImplGetReaderSchema() {
+        MessageMetadata builder = new MessageMetadata();
+        builder.hasSchemaVersion();

Review comment:
       @Shawyeok - Why was this line necessary? It seems like it's not doing anything.




-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] Shawyeok commented on pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
Shawyeok commented on pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317#issuecomment-1042679502


   Could you please take a look about this? @hangc0276 @congbobo184 


-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [pulsar] Shawyeok commented on a change in pull request #14317: [pulsar-client] Fix send to deadLetterTopic not working when reach maxRedeliverCount

Posted by GitBox <gi...@apache.org>.
Shawyeok commented on a change in pull request #14317:
URL: https://github.com/apache/pulsar/pull/14317#discussion_r813832444



##########
File path: pulsar-client/src/test/java/org/apache/pulsar/client/impl/MessageTest.java
##########
@@ -81,4 +81,15 @@ public void testTopicMessageImplNoReplicatedInfo() {
         assertFalse(topicMessage.isReplicated());
         assertNull(topicMessage.getReplicatedFrom());
     }
+
+    @Test
+    public void testMessageImplGetReaderSchema() {
+        MessageMetadata builder = new MessageMetadata();
+        builder.hasSchemaVersion();

Review comment:
       Indeed, should I open a PR to remove this line?




-- 
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: commits-unsubscribe@pulsar.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org