RobertIndie commented on code in PR #17795:
URL: https://github.com/apache/pulsar/pull/17795#discussion_r991792328


##########
pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java:
##########
@@ -519,35 +519,42 @@ public void sendAsync(Message<?> message, SendCallback 
callback) {
 
         // chunked message also sent individually so, try to acquire 
send-permits
         for (int i = 0; i < (totalChunks - 1); i++) {
-            if (!canEnqueueRequest(callback, message.getSequenceId(), 0 /* The 
memory was already reserved */)) {
+            if (!conf.isBlockIfQueueFull() && !canEnqueueRequest(callback, 
message.getSequenceId(),
+                    0 /* The memory was already reserved */)) {
                 
client.getMemoryLimitController().releaseMemory(uncompressedSize);
                 semaphoreRelease(i + 1);
                 return;
             }
         }
 
         try {
-            synchronized (this) {
-                int readStartIndex = 0;
-                String uuid = totalChunks > 1 ? String.format("%s-%d", 
producerName, sequenceId) : null;
-                ChunkedMessageCtx chunkedMessageCtx = totalChunks > 1 ? 
ChunkedMessageCtx.get(totalChunks) : null;
-                byte[] schemaVersion = totalChunks > 1 && 
msg.getMessageBuilder().hasSchemaVersion()
-                        ? msg.getMessageBuilder().getSchemaVersion() : null;
-                byte[] orderingKey = totalChunks > 1 && 
msg.getMessageBuilder().hasOrderingKey()
-                        ? msg.getMessageBuilder().getOrderingKey() : null;
-                for (int chunkId = 0; chunkId < totalChunks; chunkId++) {
-                    // Need to reset the schemaVersion, because the 
schemaVersion is based on a ByteBuf object in
-                    // `MessageMetadata`, if we want to re-serialize the 
`SEND` command using a same `MessageMetadata`,
-                    // we need to reset the ByteBuf of the schemaVersion in 
`MessageMetadata`, I think we need to
-                    // reset `ByteBuf` objects in `MessageMetadata` after call 
the method `MessageMetadata#writeTo()`.
-                    if (chunkId > 0) {
-                        if (schemaVersion != null) {
-                            
msg.getMessageBuilder().setSchemaVersion(schemaVersion);
-                        }
-                        if (orderingKey != null) {
-                            
msg.getMessageBuilder().setOrderingKey(orderingKey);
-                        }
+            int readStartIndex = 0;
+            String uuid = totalChunks > 1 ? String.format("%s-%d", 
producerName, sequenceId) : null;
+            ChunkedMessageCtx chunkedMessageCtx = totalChunks > 1 ? 
ChunkedMessageCtx.get(totalChunks) : null;
+            byte[] schemaVersion = totalChunks > 1 && 
msg.getMessageBuilder().hasSchemaVersion()
+                    ? msg.getMessageBuilder().getSchemaVersion() : null;
+            byte[] orderingKey = totalChunks > 1 && 
msg.getMessageBuilder().hasOrderingKey()
+                    ? msg.getMessageBuilder().getOrderingKey() : null;
+            for (int chunkId = 0; chunkId < totalChunks; chunkId++) {
+                // Need to reset the schemaVersion, because the schemaVersion 
is based on a ByteBuf object in
+                // `MessageMetadata`, if we want to re-serialize the `SEND` 
command using a same `MessageMetadata`,
+                // we need to reset the ByteBuf of the schemaVersion in 
`MessageMetadata`, I think we need to
+                // reset `ByteBuf` objects in `MessageMetadata` after call the 
method `MessageMetadata#writeTo()`.
+                if (chunkId > 0) {
+                    if (schemaVersion != null) {
+                        
msg.getMessageBuilder().setSchemaVersion(schemaVersion);
+                    }
+                    if (orderingKey != null) {
+                        msg.getMessageBuilder().setOrderingKey(orderingKey);
                     }
+                }
+                if (chunkId > 0 && conf.isBlockIfQueueFull() && 
!canEnqueueRequest(callback,
+                        message.getSequenceId(), 0 /* The memory was already 
reserved */)) {
+                    
client.getMemoryLimitController().releaseMemory(uncompressedSize - 
readStartIndex);
+                    semaphoreRelease(totalChunks - chunkId);
+                    return;
+                }
+                synchronized (this) {

Review Comment:
   If the thread is blocked at `canEnqueueRequest`, the IO thread will try to 
acquire this lock after receiving the ack of the send request. And this will be 
a deadlock 
[here]()https://github.com/RobertIndie/pulsar/blob/c6ebe063fc1c4088f5500ffd7ab5645c14bd581c/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java#L1126.
   
   I can also reproduce the deadlock after rebasing to the latest master branch.



-- 
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]

Reply via email to