diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java index 57fdf6f284974c..c040d2c268c9f9 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/BrokersBase.java @@ -49,7 +49,6 @@ import org.apache.pulsar.broker.admin.AdminResource; import org.apache.pulsar.broker.loadbalance.LeaderBroker; import org.apache.pulsar.broker.namespace.NamespaceService; -import org.apache.pulsar.broker.service.BrokerService; import org.apache.pulsar.broker.service.Subscription; import org.apache.pulsar.broker.service.Topic; import org.apache.pulsar.broker.web.RestException; diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java index b0e2c153899323..b943c662ba7399 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java @@ -1234,7 +1234,8 @@ private CompletableFuture> createNonPersistentTopic(String topic if (log.isDebugEnabled()) { log.debug("Broker is unable to load non-persistent topic {}", topic); } - topicFuture.completeExceptionally(new NotAllowedException("Broker is not unable to load non-persistent topic")); + topicFuture.completeExceptionally( + new NotAllowedException("Broker is not unable to load non-persistent topic")); return topicFuture; } final long topicCreateTimeMs = TimeUnit.NANOSECONDS.toMillis(System.nanoTime()); diff --git a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java index b1afe95465d318..cb6164b9888f1e 100644 --- a/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java +++ b/pulsar-client/src/main/java/org/apache/pulsar/client/impl/ProducerImpl.java @@ -322,8 +322,7 @@ CompletableFuture internalSendAsync(Message message) { interceptorMessage.getProperties(); } - int msgSize = interceptorMessage.getDataBuffer().readableBytes(); - sendAsync(interceptorMessage, new DefaultSendMessageCallback(future, interceptorMessage, msgSize)); + sendAsync(interceptorMessage, new DefaultSendMessageCallback(future, interceptorMessage)); return future; } @@ -331,15 +330,13 @@ private class DefaultSendMessageCallback implements SendCallback { CompletableFuture sendFuture; MessageImpl currentMsg; - int msgSize; long createdAt = System.nanoTime(); SendCallback nextCallback = null; MessageImpl nextMsg = null; - DefaultSendMessageCallback(CompletableFuture sendFuture, MessageImpl currentMsg, int msgSize) { + DefaultSendMessageCallback(CompletableFuture sendFuture, MessageImpl currentMsg) { this.sendFuture = sendFuture; this.currentMsg = currentMsg; - this.msgSize = msgSize; } @Override