merlimat commented on a change in pull request #1066: Issue 937: add CommandGetLastMessageId to make reader know the end of topic URL: https://github.com/apache/incubator-pulsar/pull/1066#discussion_r167382938
########## File path: pulsar-client/src/main/java/org/apache/pulsar/client/impl/ConsumerImpl.java ########## @@ -1248,6 +1254,77 @@ public void seek(MessageId messageId) throws PulsarClientException { return seekFuture; } + public boolean hasMessageAvailable() throws PulsarClientException { + try { + return hasMessageAvailableAsync().get(); + } catch (ExecutionException | InterruptedException e) { + throw new PulsarClientException(e); + } + } + + public CompletableFuture<Boolean> hasMessageAvailableAsync() { + final CompletableFuture<Boolean> booleanFuture = new CompletableFuture<>(); + + if (lastMessageIdInBroker.compareTo(lastDequeuedMessage) > 0 && + ((MessageIdImpl)lastMessageIdInBroker).getEntryId() != -1) { + booleanFuture.complete(true); + } else { + getLastMessageIdAsync().thenAccept(messageId -> { + lastMessageIdInBroker = messageId; + if (lastMessageIdInBroker.compareTo(lastDequeuedMessage) > 0 && + ((MessageIdImpl)lastMessageIdInBroker).getEntryId() != -1) { + booleanFuture.complete(true); + } else { + booleanFuture.complete(false); + } + }).exceptionally(e -> { + log.error("[{}][{}] Failed getLastMessageId command", topic, subscription); + booleanFuture.completeExceptionally(e.getCause()); + return null; + }); + } + return booleanFuture; + } + + private CompletableFuture<MessageId> getLastMessageIdAsync() { + if (getState() == State.Closing || getState() == State.Closed) { + return FutureUtil + .failedFuture(new PulsarClientException.AlreadyClosedException("Consumer was already closed")); + } + + if (cnx().getRemoteEndpointProtocolVersion() < ProtocolVersion.v11.getNumber()) { + return FutureUtil + .failedFuture(new PulsarClientException + .NotSupportedException("GetLastMessageId Not supported for ProtocolVersion: " + + cnx().getRemoteEndpointProtocolVersion())); + } + + if (!isConnected()) { Review comment: If it's currently not connected, we should try to mask the exception from the user if the failure is transient. There is already an `operationTimeout` defined in client, it would be good to have a way to retry internally with backoff up to that amount of time. ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services