-
Notifications
You must be signed in to change notification settings - Fork 3.6k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
PIP-135: Added Etcd MetadataStore implementation #13225
Conversation
@Slf4j | ||
public class EtcdMetadataStore extends AbstractBatchedMetadataStore { | ||
|
||
static final String ETCD_SCHEME_IDENTIFIER = "etcd:"; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Should it be "etcd://" to be consistent with other implementations ?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I was considering that, though it kind of gets weird with url like etcd://http://my-service:2379
. The current form of etcd:http://my-service:2379
seems a bit easier to see.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Shall we need to update the memory://
to "memory:" and "rocksdb://" to "rocksdb:"? It's better to make them consistent.
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdMetadataStore.java
Outdated
Show resolved
Hide resolved
}); | ||
|
||
if (retryOnFailure) { | ||
future.exceptionally(ex -> { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Should we return the future that only completes when retried success ?
// Re-create the lease before notifying that we are reconnected | ||
createLease(true) | ||
.thenRun(() -> { | ||
super.receivedSessionEvent(event); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
If createLease
failed the first time, the receivedSessionEvent
will not be called.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Yes, that is intended.
The code here can certainly be refactored to make it easier. The intention here is:
- At startup, we create the lease. If we fail, we bubble up the exception (eg: broker fails to start)
- If we lose the lease while running, depending on the configuration we either kill the broker or try to reconnect.
- When we don't kill the broker, we re-establish a new lease and we keep trying doing that indefinitely
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
- When we don't kill the broker, we re-establish a new lease and we keep trying doing that indefinitely
My point is, in this case, if we finally re-establish the new lease (after some retries), is it ok that we don't call super.receivedSessionEvent(event)
? Current CompletableFuture returned by createLease
only tracks the first try.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
After reconnection, the receivedSessionEvent()
will be called through the session watcher class.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I see, thx~
do we need a PIP here ? it is a big feature |
If you discount the license files, it's 1 single file containing an implementation which is not enabled by default and which does not affect anything else. I can create one.. but I don't see many design choices here to describe. |
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdSessionWatcher.java
Outdated
Show resolved
Hide resolved
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdSessionWatcher.java
Outdated
Show resolved
Hide resolved
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdSessionWatcher.java
Outdated
Show resolved
Hide resolved
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdSessionWatcher.java
Outdated
Show resolved
Hide resolved
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I believe that this will be a new feature that should be advertised and discussed on dev@
This is why I am thinking about starting a PIP.
I am pretty sure that no one will object to committing this patch, but for the benefit of the many folks in the community that are not following the github pull requests flow it is better to advertise about this work.
I will do my review when I wil be back form vacation (no need to wait for me, for sure)
@Slf4j | ||
public class EtcdMetadataStore extends AbstractBatchedMetadataStore { | ||
|
||
static final String ETCD_SCHEME_IDENTIFIER = "etcd:"; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Shall we need to update the memory://
to "memory:" and "rocksdb://" to "rocksdb:"? It's better to make them consistent.
} | ||
} | ||
|
||
private synchronized CompletableFuture<Void> createLease(boolean retryOnFailure) { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Does retryOnFailure
is to retry in the background but the caller will get an exception?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
No, this is after we created the lease the first time, so that we can re-create it in background. If we fail, we notify the session listener to change the current state and we keep retrying in background.
pulsar-metadata/src/main/java/org/apache/pulsar/metadata/impl/EtcdMetadataStore.java
Outdated
Show resolved
Hide resolved
Good point, we should do that in a different PR. |
@Override | ||
protected CompletableFuture<Boolean> existsFromStore(String path) { | ||
return kv.get(ByteSequence.from(path, StandardCharsets.UTF_8), EXISTS_GET_OPTION) | ||
.thenApply(gr -> gr.getCount() == 1); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Do we need to ensure the async callback is executed in metadata-store
thread.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
👍 Fixed it.
Using the consistent metadata store scheme name. - RocksDB -> rocksdb: - ZooKeeper -> zk: - Memory -> memory: - Ectd -> etcd: Context: #13225 (comment)
Using the consistent metadata store scheme name. - RocksDB -> rocksdb: - ZooKeeper -> zk: - Memory -> memory: - Ectd -> etcd: Context: apache#13225 (comment)
Motivation
Added a new MetadataStore implementation based on Etcd client.
The new implementation has all the feature as the ZK based backend: