Skip to content

Commit c172a77

Browse files
authored
[fix][broker] Fix can't send ErrorCommand when message is null value (apache#19899)
1 parent a1fdbd2 commit c172a77

File tree

2 files changed

+28
-5
lines changed

2 files changed

+28
-5
lines changed

pulsar-broker/src/test/java/org/apache/pulsar/client/api/ClientErrorsTest.java

Lines changed: 26 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -22,16 +22,15 @@
2222
import static org.testng.Assert.assertFalse;
2323
import static org.testng.Assert.assertTrue;
2424
import static org.testng.Assert.fail;
25-
2625
import io.netty.channel.ChannelHandlerContext;
27-
26+
import java.util.Map;
2827
import java.util.concurrent.CountDownLatch;
2928
import java.util.concurrent.TimeUnit;
3029
import java.util.concurrent.atomic.AtomicBoolean;
3130
import java.util.concurrent.atomic.AtomicInteger;
3231
import java.util.concurrent.atomic.AtomicReference;
33-
3432
import lombok.Cleanup;
33+
import org.apache.bookkeeper.common.util.JsonUtil;
3534
import org.apache.pulsar.client.impl.ConsumerBase;
3635
import org.apache.pulsar.client.impl.PartitionedProducerImpl;
3736
import org.apache.pulsar.client.impl.ProducerBase;
@@ -831,4 +830,28 @@ public void testConsumerReconnect() throws Exception {
831830
mockBrokerService.resetHandleConnect();
832831
mockBrokerService.resetHandleSubscribe();
833832
}
833+
834+
@Test
835+
public void testCommandErrorMessageIsNull() throws Exception {
836+
@Cleanup
837+
PulsarClient client = PulsarClient.builder().serviceUrl(mockBrokerService.getBrokerAddress()).build();
838+
839+
mockBrokerService.setHandleProducer((ctx, producer) -> {
840+
try {
841+
ctx.writeAndFlush(Commands.newError(producer.getRequestId(), ServerError.AuthorizationError, null));
842+
} catch (Exception e) {
843+
fail("Send error command failed", e);
844+
}
845+
});
846+
847+
try {
848+
client.newProducer().topic("persistent://prop/use/ns/t1").create();
849+
fail();
850+
} catch (Exception e) {
851+
assertTrue(e instanceof PulsarClientException.AuthorizationException);
852+
Map<String, String> map = JsonUtil.fromJson(e.getMessage(), Map.class);
853+
assertEquals(map.get("errorMsg"), "");
854+
}
855+
mockBrokerService.resetHandleProducer();
856+
}
834857
}

pulsar-common/src/main/java/org/apache/pulsar/common/protocol/Commands.java

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -368,7 +368,7 @@ public static BaseCommand newErrorCommand(long requestId, ServerError serverErro
368368
cmd.setError()
369369
.setRequestId(requestId)
370370
.setError(serverError)
371-
.setMessage(message);
371+
.setMessage(message != null ? message : "");
372372
return cmd;
373373
}
374374

@@ -401,7 +401,7 @@ public static BaseCommand newSendErrorCommand(long producerId, long sequenceId,
401401
.setProducerId(producerId)
402402
.setSequenceId(sequenceId)
403403
.setError(error)
404-
.setMessage(errorMsg);
404+
.setMessage(errorMsg != null ? errorMsg : "");
405405
return cmd;
406406
}
407407

0 commit comments

Comments
 (0)