@@ -167,7 +167,9 @@ public class ConsumerImpl<T> extends ConsumerBase<T> implements ConnectionHandle
167
167
private volatile MessageIdAdv startMessageId ;
168
168
169
169
private volatile MessageIdAdv seekMessageId ;
170
- private final AtomicBoolean duringSeek ;
170
+ @ VisibleForTesting
171
+ final AtomicReference <SeekStatus > seekStatus ;
172
+ private volatile CompletableFuture <Void > seekFuture ;
171
173
172
174
private final MessageIdAdv initialStartMessageId ;
173
175
@@ -304,7 +306,7 @@ protected ConsumerImpl(PulsarClientImpl client, String topic, ConsumerConfigurat
304
306
stats = ConsumerStatsDisabled .INSTANCE ;
305
307
}
306
308
307
- duringSeek = new AtomicBoolean ( false );
309
+ seekStatus = new AtomicReference <>( SeekStatus . NOT_STARTED );
308
310
309
311
// Create msgCrypto if not created already
310
312
if (conf .getCryptoKeyReader () != null ) {
@@ -781,15 +783,15 @@ public CompletableFuture<Void> connectionOpened(final ClientCnx cnx) {
781
783
closeConsumerTasks ();
782
784
deregisterFromClientCnx ();
783
785
client .cleanupConsumer (this );
784
- clearReceiverQueue ();
786
+ clearReceiverQueue (false );
785
787
return CompletableFuture .completedFuture (null );
786
788
}
787
789
788
790
log .info ("[{}][{}] Subscribing to topic on cnx {}, consumerId {}" ,
789
791
topic , subscription , cnx .ctx ().channel (), consumerId );
790
792
791
793
long requestId = client .newRequestId ();
792
- if (duringSeek .get ()) {
794
+ if (seekStatus .get () != SeekStatus . NOT_STARTED ) {
793
795
acknowledgmentsGroupingTracker .flushAndClean ();
794
796
}
795
797
@@ -800,7 +802,8 @@ public CompletableFuture<Void> connectionOpened(final ClientCnx cnx) {
800
802
int currentSize ;
801
803
synchronized (this ) {
802
804
currentSize = incomingMessages .size ();
803
- startMessageId = clearReceiverQueue ();
805
+ setClientCnx (cnx );
806
+ clearReceiverQueue (true );
804
807
if (possibleSendToDeadLetterTopicMessages != null ) {
805
808
possibleSendToDeadLetterTopicMessages .clear ();
806
809
}
@@ -838,7 +841,6 @@ public CompletableFuture<Void> connectionOpened(final ClientCnx cnx) {
838
841
// synchronized this, because redeliverUnAckMessage eliminate the epoch inconsistency between them
839
842
final CompletableFuture <Void > future = new CompletableFuture <>();
840
843
synchronized (this ) {
841
- setClientCnx (cnx );
842
844
ByteBuf request = Commands .newSubscribe (topic , subscription , consumerId , requestId , getSubType (),
843
845
priorityLevel , consumerName , isDurable , startMessageIdData , metadata , readCompacted ,
844
846
conf .isReplicateSubscriptionState (),
@@ -943,15 +945,24 @@ protected void consumerIsReconnectedToBroker(ClientCnx cnx, int currentQueueSize
943
945
* Clear the internal receiver queue and returns the message id of what was the 1st message in the queue that was
944
946
* not seen by the application.
945
947
*/
946
- private MessageIdAdv clearReceiverQueue () {
948
+ private void clearReceiverQueue (boolean updateStartMessageId ) {
947
949
List <Message <?>> currentMessageQueue = new ArrayList <>(incomingMessages .size ());
948
950
incomingMessages .drainTo (currentMessageQueue );
949
951
resetIncomingMessageSize ();
950
952
951
- if (duringSeek .compareAndSet (true , false )) {
952
- return seekMessageId ;
953
+ CompletableFuture <Void > seekFuture = this .seekFuture ;
954
+ MessageIdAdv seekMessageId = this .seekMessageId ;
955
+
956
+ if (seekStatus .get () != SeekStatus .NOT_STARTED ) {
957
+ if (updateStartMessageId ) {
958
+ startMessageId = seekMessageId ;
959
+ }
960
+ if (seekStatus .compareAndSet (SeekStatus .COMPLETED , SeekStatus .NOT_STARTED )) {
961
+ internalPinnedExecutor .execute (() -> seekFuture .complete (null ));
962
+ }
963
+ return ;
953
964
} else if (subscriptionMode == SubscriptionMode .Durable ) {
954
- return startMessageId ;
965
+ return ;
955
966
}
956
967
957
968
if (!currentMessageQueue .isEmpty ()) {
@@ -968,15 +979,14 @@ private MessageIdAdv clearReceiverQueue() {
968
979
}
969
980
// release messages if they are pooled messages
970
981
currentMessageQueue .forEach (Message ::release );
971
- return previousMessage ;
972
- } else if (!lastDequeuedMessageId .equals (MessageId .earliest )) {
982
+ if (updateStartMessageId ) {
983
+ startMessageId = previousMessage ;
984
+ }
985
+ } else if (updateStartMessageId && !lastDequeuedMessageId .equals (MessageId .earliest )) {
973
986
// If the queue was empty we need to restart from the message just after the last one that has been dequeued
974
987
// in the past
975
- return new BatchMessageIdImpl ((MessageIdImpl ) lastDequeuedMessageId );
976
- } else {
977
- // No message was received or dequeued by this consumer. Next message would still be the startMessageId
978
- return startMessageId ;
979
- }
988
+ startMessageId = new BatchMessageIdImpl ((MessageIdImpl ) lastDequeuedMessageId );
989
+ } // else: No message was received or dequeued by this consumer. Next message would still be the startMessageId
980
990
}
981
991
982
992
/**
@@ -2249,25 +2259,23 @@ private CompletableFuture<Void> seekAsyncInternal(long requestId, ByteBuf seek,
2249
2259
.setMandatoryStop (0 , TimeUnit .MILLISECONDS )
2250
2260
.create ();
2251
2261
2252
- CompletableFuture <Void > seekFuture = new CompletableFuture <>();
2253
- seekAsyncInternal (requestId , seek , seekId , seekBy , backoff , opTimeoutMs , seekFuture );
2262
+ if (!seekStatus .compareAndSet (SeekStatus .NOT_STARTED , SeekStatus .IN_PROGRESS )) {
2263
+ final String message = String .format (
2264
+ "[%s][%s] attempting to seek operation that is already in progress (seek by %s)" ,
2265
+ topic , subscription , seekBy );
2266
+ log .warn ("[{}][{}] Attempting to seek operation that is already in progress, cancelling {}" ,
2267
+ topic , subscription , seekBy );
2268
+ return FutureUtil .failedFuture (new IllegalStateException (message ));
2269
+ }
2270
+ seekFuture = new CompletableFuture <>();
2271
+ seekAsyncInternal (requestId , seek , seekId , seekBy , backoff , opTimeoutMs );
2254
2272
return seekFuture ;
2255
2273
}
2256
2274
2257
2275
private void seekAsyncInternal (long requestId , ByteBuf seek , MessageId seekId , String seekBy ,
2258
- final Backoff backoff , final AtomicLong remainingTime ,
2259
- CompletableFuture <Void > seekFuture ) {
2276
+ final Backoff backoff , final AtomicLong remainingTime ) {
2260
2277
ClientCnx cnx = cnx ();
2261
2278
if (isConnected () && cnx != null ) {
2262
- if (!duringSeek .compareAndSet (false , true )) {
2263
- final String message = String .format (
2264
- "[%s][%s] attempting to seek operation that is already in progress (seek by %s)" ,
2265
- topic , subscription , seekBy );
2266
- log .warn ("[{}][{}] Attempting to seek operation that is already in progress, cancelling {}" ,
2267
- topic , subscription , seekBy );
2268
- seekFuture .completeExceptionally (new IllegalStateException (message ));
2269
- return ;
2270
- }
2271
2279
MessageIdAdv originSeekMessageId = seekMessageId ;
2272
2280
seekMessageId = (MessageIdAdv ) seekId ;
2273
2281
log .info ("[{}][{}] Seeking subscription to {}" , topic , subscription , seekBy );
@@ -2279,14 +2287,25 @@ private void seekAsyncInternal(long requestId, ByteBuf seek, MessageId seekId, S
2279
2287
lastDequeuedMessageId = MessageId .earliest ;
2280
2288
2281
2289
clearIncomingMessages ();
2282
- seekFuture .complete (null );
2290
+ CompletableFuture <Void > future = null ;
2291
+ synchronized (this ) {
2292
+ if (!hasParentConsumer && cnx () == null ) {
2293
+ // It's during reconnection, complete the seek future after connection is established
2294
+ seekStatus .set (SeekStatus .COMPLETED );
2295
+ } else {
2296
+ future = seekFuture ;
2297
+ startMessageId = seekMessageId ;
2298
+ seekStatus .set (SeekStatus .NOT_STARTED );
2299
+ }
2300
+ }
2301
+ if (future != null ) {
2302
+ future .complete (null );
2303
+ }
2283
2304
}).exceptionally (e -> {
2284
- // re-set duringSeek and seekMessageId if seek failed
2285
2305
seekMessageId = originSeekMessageId ;
2286
- duringSeek .set (false );
2287
2306
log .error ("[{}][{}] Failed to reset subscription: {}" , topic , subscription , e .getCause ().getMessage ());
2288
2307
2289
- seekFuture . completeExceptionally (
2308
+ failSeek (
2290
2309
PulsarClientException .wrap (e .getCause (),
2291
2310
String .format ("Failed to seek the subscription %s of the topic %s to %s" ,
2292
2311
subscription , topicName .toString (), seekBy )));
@@ -2295,7 +2314,7 @@ private void seekAsyncInternal(long requestId, ByteBuf seek, MessageId seekId, S
2295
2314
} else {
2296
2315
long nextDelay = Math .min (backoff .next (), remainingTime .get ());
2297
2316
if (nextDelay <= 0 ) {
2298
- seekFuture . completeExceptionally (
2317
+ failSeek (
2299
2318
new PulsarClientException .TimeoutException (
2300
2319
String .format ("The subscription %s of the topic %s could not seek "
2301
2320
+ "withing configured timeout" , subscription , topicName .toString ())));
@@ -2306,11 +2325,18 @@ private void seekAsyncInternal(long requestId, ByteBuf seek, MessageId seekId, S
2306
2325
log .warn ("[{}] [{}] Could not get connection while seek -- Will try again in {} ms" ,
2307
2326
topic , getHandlerName (), nextDelay );
2308
2327
remainingTime .addAndGet (-nextDelay );
2309
- seekAsyncInternal (requestId , seek , seekId , seekBy , backoff , remainingTime , seekFuture );
2328
+ seekAsyncInternal (requestId , seek , seekId , seekBy , backoff , remainingTime );
2310
2329
}, nextDelay , TimeUnit .MILLISECONDS );
2311
2330
}
2312
2331
}
2313
2332
2333
+ private void failSeek (Throwable throwable ) {
2334
+ CompletableFuture <Void > seekFuture = this .seekFuture ;
2335
+ if (seekStatus .compareAndSet (SeekStatus .IN_PROGRESS , SeekStatus .NOT_STARTED )) {
2336
+ seekFuture .completeExceptionally (throwable );
2337
+ }
2338
+ }
2339
+
2314
2340
@ Override
2315
2341
public CompletableFuture <Void > seekAsync (long timestamp ) {
2316
2342
String seekBy = String .format ("the timestamp %d" , timestamp );
@@ -2968,4 +2994,10 @@ boolean isAckReceiptEnabled() {
2968
2994
2969
2995
private static final Logger log = LoggerFactory .getLogger (ConsumerImpl .class );
2970
2996
2997
+ @ VisibleForTesting
2998
+ enum SeekStatus {
2999
+ NOT_STARTED ,
3000
+ IN_PROGRESS ,
3001
+ COMPLETED
3002
+ }
2971
3003
}
0 commit comments