Skip to content

Commit 2d89341

Browse files
authored
feat: Ability to reset subscriber upon out of band seek (#662)
Prepares for handling the RESET signal from the server, by: - Discarding outstanding acks for delivered messages. - Waiting for the committer to flush pending commits and receive the acknowledgment from the server. - Then resetting subscriber state, including canceling any in-flight client seeks.
1 parent 9e91ecd commit 2d89341

19 files changed

+419
-54
lines changed

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/cloudpubsub/SubscriberSettings.java

Lines changed: 5 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -260,7 +260,11 @@ Subscriber newPartitionSubscriber(Partition partition) throws CheckedApiExceptio
260260
partition, transformer().orElse(MessageTransforms.toCpsSubscribeTransformer())),
261261
new AckSetTrackerImpl(wireCommitter),
262262
nackHandler().orElse(new NackHandler() {}),
263-
messageConsumer -> wireSubscriberBuilder.setMessageConsumer(messageConsumer).build(),
263+
(messageConsumer, resetHandler) ->
264+
wireSubscriberBuilder
265+
.setMessageConsumer(messageConsumer)
266+
.setResetHandler(resetHandler)
267+
.build(),
264268
perPartitionFlowControlSettings());
265269
} catch (Throwable t) {
266270
throw toCanonical(t);

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/cloudpubsub/internal/AckSetTracker.java

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -24,4 +24,8 @@ interface AckSetTracker extends ApiService {
2424
// Track the given message. Returns a Runnable to ack this message if the message is a valid one
2525
// to add to the ack set. Must be called with strictly increasing offset messages.
2626
Runnable track(SequencedMessage message) throws CheckedApiException;
27+
28+
// Discard all outstanding acks and wait for any pending commit offset to be acknowledged by the
29+
// server. Throws an exception if the committer shut down due to a permanent error.
30+
void waitUntilCommitted() throws CheckedApiException;
2731
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/cloudpubsub/internal/AckSetTrackerImpl.java

Lines changed: 64 additions & 18 deletions
Original file line numberDiff line numberDiff line change
@@ -28,21 +28,63 @@
2828
import com.google.cloud.pubsublite.internal.ExtractStatus;
2929
import com.google.cloud.pubsublite.internal.TrivialProxyService;
3030
import com.google.cloud.pubsublite.internal.wire.Committer;
31+
import com.google.common.collect.ImmutableList;
3132
import com.google.errorprone.annotations.concurrent.GuardedBy;
3233
import java.util.ArrayDeque;
3334
import java.util.Deque;
35+
import java.util.List;
3436
import java.util.Optional;
3537
import java.util.PriorityQueue;
36-
import java.util.concurrent.atomic.AtomicBoolean;
3738

3839
public class AckSetTrackerImpl extends TrivialProxyService implements AckSetTracker {
40+
// Receipt represents an unacked message. It can be cleared, which will cause the ack to be
41+
// ignored.
42+
private static class Receipt {
43+
final Offset offset;
44+
45+
private final CloseableMonitor m = new CloseableMonitor();
46+
47+
@GuardedBy("m.monitor")
48+
private boolean wasAcked = false;
49+
50+
@GuardedBy("m.monitor")
51+
private Optional<AckSetTrackerImpl> tracker;
52+
53+
Receipt(Offset offset, AckSetTrackerImpl tracker) {
54+
this.offset = offset;
55+
this.tracker = Optional.of(tracker);
56+
}
57+
58+
void clear() {
59+
try (CloseableMonitor.Hold h = m.enter()) {
60+
tracker = Optional.empty();
61+
}
62+
}
63+
64+
void onAck() {
65+
try (CloseableMonitor.Hold h = m.enter()) {
66+
if (!tracker.isPresent()) {
67+
return;
68+
}
69+
if (wasAcked) {
70+
CheckedApiException e =
71+
new CheckedApiException("Duplicate acks are not allowed.", Code.FAILED_PRECONDITION);
72+
tracker.get().onPermanentError(e);
73+
throw e.underlying;
74+
}
75+
wasAcked = true;
76+
tracker.get().onAck(offset);
77+
}
78+
}
79+
}
80+
3981
private final CloseableMonitor monitor = new CloseableMonitor();
4082

4183
@GuardedBy("monitor.monitor")
4284
private final Committer committer;
4385

4486
@GuardedBy("monitor.monitor")
45-
private final Deque<Offset> receipts = new ArrayDeque<>();
87+
private final Deque<Receipt> receipts = new ArrayDeque<>();
4688

4789
@GuardedBy("monitor.monitor")
4890
private final PriorityQueue<Offset> acks = new PriorityQueue<>();
@@ -57,23 +99,27 @@ public AckSetTrackerImpl(Committer committer) throws ApiException {
5799
public Runnable track(SequencedMessage message) throws CheckedApiException {
58100
final Offset messageOffset = message.offset();
59101
try (CloseableMonitor.Hold h = monitor.enter()) {
60-
checkArgument(receipts.isEmpty() || receipts.peekLast().value() < messageOffset.value());
61-
receipts.addLast(messageOffset);
102+
checkArgument(
103+
receipts.isEmpty() || receipts.peekLast().offset.value() < messageOffset.value());
104+
Receipt receipt = new Receipt(messageOffset, this);
105+
receipts.addLast(receipt);
106+
return receipt::onAck;
62107
}
63-
return new Runnable() {
64-
private final AtomicBoolean wasAcked = new AtomicBoolean(false);
108+
}
65109

66-
@Override
67-
public void run() {
68-
if (wasAcked.getAndSet(true)) {
69-
CheckedApiException e =
70-
new CheckedApiException("Duplicate acks are not allowed.", Code.FAILED_PRECONDITION);
71-
onPermanentError(e);
72-
throw e.underlying;
73-
}
74-
onAck(messageOffset);
75-
}
76-
};
110+
@Override
111+
public void waitUntilCommitted() throws CheckedApiException {
112+
List<Receipt> receiptsCopy;
113+
try (CloseableMonitor.Hold h = monitor.enter()) {
114+
receiptsCopy = ImmutableList.copyOf(receipts);
115+
}
116+
// Clearing receipts here avoids deadlocks due to locks acquired in different order.
117+
receiptsCopy.forEach(Receipt::clear);
118+
try (CloseableMonitor.Hold h = monitor.enter()) {
119+
receipts.clear();
120+
acks.clear();
121+
committer.waitUntilEmpty();
122+
}
77123
}
78124

79125
private void onAck(Offset offset) {
@@ -82,7 +128,7 @@ private void onAck(Offset offset) {
82128
Optional<Offset> prefixAckedOffset = Optional.empty();
83129
while (!receipts.isEmpty()
84130
&& !acks.isEmpty()
85-
&& receipts.peekFirst().value() == acks.peek().value()) {
131+
&& receipts.peekFirst().offset.value() == acks.peek().value()) {
86132
prefixAckedOffset = Optional.of(acks.remove());
87133
receipts.removeFirst();
88134
}
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,32 @@
1+
/*
2+
* Copyright 2021 Google LLC
3+
*
4+
* Licensed under the Apache License, Version 2.0 (the "License");
5+
* you may not use this file except in compliance with the License.
6+
* You may obtain a copy of the License at
7+
*
8+
* http://www.apache.org/licenses/LICENSE-2.0
9+
*
10+
* Unless required by applicable law or agreed to in writing, software
11+
* distributed under the License is distributed on an "AS IS" BASIS,
12+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
13+
* See the License for the specific language governing permissions and
14+
* limitations under the License.
15+
*/
16+
17+
package com.google.cloud.pubsublite.cloudpubsub.internal;
18+
19+
import com.google.api.gax.rpc.ApiException;
20+
import com.google.cloud.pubsublite.SequencedMessage;
21+
import com.google.cloud.pubsublite.internal.wire.Subscriber;
22+
import com.google.cloud.pubsublite.internal.wire.SubscriberResetHandler;
23+
import com.google.common.collect.ImmutableList;
24+
import java.io.Serializable;
25+
import java.util.function.Consumer;
26+
27+
public interface ResettableSubscriberFactory extends Serializable {
28+
Subscriber newSubscriber(
29+
Consumer<ImmutableList<SequencedMessage>> messageConsumer,
30+
SubscriberResetHandler resetHandler)
31+
throws ApiException;
32+
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/cloudpubsub/internal/SinglePartitionSubscriber.java

Lines changed: 9 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -30,7 +30,6 @@
3030
import com.google.cloud.pubsublite.internal.CheckedApiException;
3131
import com.google.cloud.pubsublite.internal.ExtractStatus;
3232
import com.google.cloud.pubsublite.internal.ProxyService;
33-
import com.google.cloud.pubsublite.internal.wire.SubscriberFactory;
3433
import com.google.cloud.pubsublite.proto.FlowControlRequest;
3534
import com.google.common.annotations.VisibleForTesting;
3635
import com.google.common.collect.ImmutableList;
@@ -50,15 +49,16 @@ public SinglePartitionSubscriber(
5049
MessageTransformer<SequencedMessage, PubsubMessage> transformer,
5150
AckSetTracker ackSetTracker,
5251
NackHandler nackHandler,
53-
SubscriberFactory wireSubscriberFactory,
52+
ResettableSubscriberFactory wireSubscriberFactory,
5453
FlowControlSettings flowControlSettings)
5554
throws ApiException {
5655
this.receiver = receiver;
5756
this.transformer = transformer;
5857
this.ackSetTracker = ackSetTracker;
5958
this.nackHandler = nackHandler;
6059
this.flowControlSettings = flowControlSettings;
61-
this.wireSubscriber = wireSubscriberFactory.newSubscriber(this::onMessages);
60+
this.wireSubscriber =
61+
wireSubscriberFactory.newSubscriber(this::onMessages, this::onSubscriberReset);
6262
addServices(ackSetTracker, wireSubscriber);
6363
}
6464

@@ -126,4 +126,10 @@ public void onSuccess(Void result) {
126126
onPermanentError(ExtractStatus.toCanonical(t));
127127
}
128128
}
129+
130+
@VisibleForTesting
131+
boolean onSubscriberReset() throws CheckedApiException {
132+
// TODO: handle reset.
133+
return false;
134+
}
129135
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/internal/wire/ApiExceptionCommitter.java

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@
2121
import com.google.api.core.ApiFuture;
2222
import com.google.api.gax.rpc.ApiException;
2323
import com.google.cloud.pubsublite.Offset;
24+
import com.google.cloud.pubsublite.internal.CheckedApiException;
2425
import com.google.cloud.pubsublite.internal.TrivialProxyService;
2526

2627
class ApiExceptionCommitter extends TrivialProxyService implements Committer {
@@ -35,4 +36,9 @@ class ApiExceptionCommitter extends TrivialProxyService implements Committer {
3536
public ApiFuture<Void> commitOffset(Offset offset) {
3637
return toClientFuture(committer.commitOffset(offset));
3738
}
39+
40+
@Override
41+
public void waitUntilEmpty() throws CheckedApiException {
42+
committer.waitUntilEmpty();
43+
}
3844
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/internal/wire/CommitState.java

Lines changed: 7 additions & 9 deletions
Original file line numberDiff line numberDiff line change
@@ -59,15 +59,13 @@ ApiFuture<Void> addCommit(Offset offset) {
5959

6060
void complete(long numComplete) throws CheckedApiException {
6161
if (numComplete > currentConnectionFutures.size()) {
62-
CheckedApiException error =
63-
new CheckedApiException(
64-
String.format(
65-
"Received %s completions, which is more than the commits outstanding for this"
66-
+ " stream.",
67-
numComplete),
68-
Code.FAILED_PRECONDITION);
69-
abort(error);
70-
throw error;
62+
// Note: Throw here to permanently shut down CommitterImpl, which will later call abort().
63+
throw new CheckedApiException(
64+
String.format(
65+
"Received %s completions, which is more than the commits outstanding for this"
66+
+ " stream.",
67+
numComplete),
68+
Code.FAILED_PRECONDITION);
7169
}
7270
while (!pastConnectionFutures.isEmpty()) {
7371
// Past futures refer to commits sent chronologically before the current stream, and thus they

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/internal/wire/Committer.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -19,8 +19,13 @@
1919
import com.google.api.core.ApiFuture;
2020
import com.google.api.core.ApiService;
2121
import com.google.cloud.pubsublite.Offset;
22+
import com.google.cloud.pubsublite.internal.CheckedApiException;
2223

2324
public interface Committer extends ApiService {
2425
// Commit a given offset. Clean shutdown waits for all outstanding commits to complete.
2526
ApiFuture<Void> commitOffset(Offset offset);
27+
28+
// Waits until all commits have been sent and acknowledged by the server. Throws an exception if
29+
// the committer shut down due to a permanent error.
30+
void waitUntilEmpty() throws CheckedApiException;
2631
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/internal/wire/CommitterImpl.java

Lines changed: 12 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -46,7 +46,7 @@ public class CommitterImpl extends ProxyService
4646
new Guard(monitor.monitor) {
4747
public boolean isSatisfied() {
4848
// Wait until the state is empty or a permanent error occurred.
49-
return state.isEmpty() || hadPermanentError;
49+
return state.isEmpty() || permanentError.isPresent();
5050
}
5151
};
5252

@@ -57,7 +57,7 @@ public boolean isSatisfied() {
5757
private boolean shutdown = false;
5858

5959
@GuardedBy("monitor.monitor")
60-
private boolean hadPermanentError = false;
60+
private Optional<CheckedApiException> permanentError = Optional.empty();
6161

6262
@GuardedBy("monitor.monitor")
6363
private final CommitState state = new CommitState();
@@ -88,7 +88,7 @@ public CommitterImpl(CursorServiceClient client, InitialCommitCursorRequest requ
8888
@Override
8989
protected void handlePermanentError(CheckedApiException error) {
9090
try (CloseableMonitor.Hold h = monitor.enter()) {
91-
hadPermanentError = true;
91+
permanentError = Optional.of(error);
9292
shutdown = true;
9393
state.abort(error);
9494
}
@@ -144,4 +144,13 @@ public ApiFuture<Void> commitOffset(Offset offset) {
144144
return ApiFutures.immediateFailedFuture(e);
145145
}
146146
}
147+
148+
@Override
149+
public void waitUntilEmpty() throws CheckedApiException {
150+
try (CloseableMonitor.Hold h = monitor.enterWhenUninterruptibly(isEmptyOrError)) {
151+
if (permanentError.isPresent()) {
152+
throw permanentError.get();
153+
}
154+
}
155+
}
147156
}

google-cloud-pubsublite/src/main/java/com/google/cloud/pubsublite/internal/wire/NextOffsetTracker.java

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -59,4 +59,9 @@ Optional<SeekRequest> requestForRestart() {
5959
.setCursor(Cursor.newBuilder().setOffset(offset.value()))
6060
.build());
6161
}
62+
63+
// Resets the offset tracker to its initial state.
64+
void reset() {
65+
nextOffset = Optional.empty();
66+
}
6267
}

0 commit comments

Comments
 (0)