-
Notifications
You must be signed in to change notification settings - Fork 3.8k
Zookeeper loss #6740
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
Zookeeper loss #6740
Changes from all commits
5edc6da
b9e8650
c8ce78c
8b64673
39d538d
0df7274
1ece697
3fb949e
1e7774e
2d16309
1ecd489
4459257
d9187e0
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
| @@ -0,0 +1,63 @@ | ||
| /* | ||
| * Licensed to the Apache Software Foundation (ASF) under one | ||
| * or more contributor license agreements. See the NOTICE file | ||
| * distributed with this work for additional information | ||
| * regarding copyright ownership. The ASF licenses this file | ||
| * to you under the Apache License, Version 2.0 (the | ||
| * "License"); you may not use this file except in compliance | ||
| * with the License. You may obtain a copy of the License at | ||
| * | ||
| * http://www.apache.org/licenses/LICENSE-2.0 | ||
| * | ||
| * Unless required by applicable law or agreed to in writing, | ||
| * software distributed under the License is distributed on an | ||
| * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY | ||
| * KIND, either express or implied. See the License for the | ||
| * specific language governing permissions and limitations | ||
| * under the License. | ||
| */ | ||
|
|
||
| package org.apache.druid.curator; | ||
|
|
||
| import org.apache.curator.RetrySleeper; | ||
| import org.apache.curator.retry.BoundedExponentialBackoffRetry; | ||
| import org.apache.druid.java.util.common.logger.Logger; | ||
|
|
||
| /** | ||
| * BoundedExponentialBackoffRetryWithQuit extends BoundedExponentialBackoffRetry for simplicity. It's not actually a | ||
| * BoundedExponentialBackoffRetry from the Liskov substitution principle point of view, | ||
| * but it doesn't matter in this code. | ||
| * | ||
| */ | ||
| public class BoundedExponentialBackoffRetryWithQuit extends BoundedExponentialBackoffRetry | ||
| { | ||
|
|
||
| private static final Logger log = new Logger(BoundedExponentialBackoffRetryWithQuit.class); | ||
|
|
||
| private final Runnable exitRunner; | ||
|
|
||
| public BoundedExponentialBackoffRetryWithQuit( | ||
| Runnable exitRunner, | ||
| int baseSleepTimeMs, | ||
| int maxSleepTimeMs, | ||
| int maxRetries | ||
| ) | ||
| { | ||
| super(baseSleepTimeMs, maxSleepTimeMs, maxRetries); | ||
| this.exitRunner = exitRunner; | ||
| log.info("BoundedExponentialBackoffRetryWithQuit Retry Policy selected."); | ||
| } | ||
|
|
||
| @Override | ||
| public boolean allowRetry(int retryCount, long elapsedTimeMs, RetrySleeper sleeper) | ||
| { | ||
| log.warn("Zookeeper can't be reached, retrying (retryCount = %s out of %s)...", retryCount, this.getN()); | ||
| boolean shouldRetry = super.allowRetry(retryCount, elapsedTimeMs, sleeper); | ||
| if (!shouldRetry) { | ||
| log.warn("Since Zookeeper can't be reached after retries exhausted, calling exit function..."); | ||
| exitRunner.run(); | ||
| } | ||
| return shouldRetry; | ||
| } | ||
|
|
||
| } | ||
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -22,6 +22,8 @@ | |
| import com.google.inject.Binder; | ||
| import com.google.inject.Module; | ||
| import com.google.inject.Provides; | ||
| import io.netty.util.SuppressForbidden; | ||
| import org.apache.curator.RetryPolicy; | ||
| import org.apache.curator.ensemble.EnsembleProvider; | ||
| import org.apache.curator.ensemble.exhibitor.DefaultExhibitorRestClient; | ||
| import org.apache.curator.ensemble.exhibitor.ExhibitorEnsembleProvider; | ||
|
|
@@ -70,6 +72,7 @@ public void configure(Binder binder) | |
|
|
||
| @Provides | ||
| @LazySingleton | ||
| @SuppressForbidden(reason = "System#err") | ||
| public CuratorFramework makeCurator(CuratorConfig config, EnsembleProvider ensembleProvider, Lifecycle lifecycle) | ||
| { | ||
| final Builder builder = CuratorFrameworkFactory.builder(); | ||
|
|
@@ -79,10 +82,33 @@ public CuratorFramework makeCurator(CuratorConfig config, EnsembleProvider ensem | |
| StringUtils.format("%s:%s", config.getZkUser(), config.getZkPwd()).getBytes(StandardCharsets.UTF_8) | ||
| ); | ||
| } | ||
|
|
||
| RetryPolicy retryPolicy; | ||
| if (config.getTerminateDruidProcessOnConnectFail()) { | ||
| final Runnable exitRunner = () -> { | ||
| try { | ||
| log.error("Zookeeper can't be reached, forcefully stopping lifecycle..."); | ||
| lifecycle.stop(); | ||
| System.err.println("Zookeeper can't be reached, forcefully stopping virtual machine..."); | ||
| } | ||
| finally { | ||
| System.exit(1); | ||
|
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. If you really want to exit, try {
log.error("Zookeeper can't be reached, forcefully stopping lifecycle...");
// Maybe wrap with call and log exception too
lifecycle.stop();
log.error("Zookeeper can't be reached, forcefully stopping virtual machine...");
} finally {
System.exit(1);
}would be more robust
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. For me it looks like a hack, shouldn't we try to recreate a CuratorFramework?
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Correct me if I'm wrong, please, we could have the whole Druid clusters restarted due to a ZK cluster shortage?
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
@egor-ryashin That's the point. It is disabled by default. A lot of us want to have the Java process die (so that an outside process can restart it, like other popular Linux daemons). Some scenarios cause it to not completely die, because a Zookeeper thread hangs on forever. Our use case was this: DNS/IT services changed the IPs of the zookeeper cluster after the JVM had already cached them. Had they been able to die gracefully, a daemon would have caused the cluster to "self heal." This use case is not an exhaustive list. Also, we noticed that queries gave bad data during these times. Sometimes, for some use cases (as is ours), no data is preferable to bad data.
This would not have fixed the issue in our use case.
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
I wonder whether using the option
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. It probably can. However, all foreseeable use cases cannot be determined at this time.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Another one may have presented itself today. I contend that having this in there would be a net positive. https://groups.google.com/forum/#!topic/druid-user/ZK0VMaEIM1w
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I believe you cannot write a unit-test for that.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. You're basically right. Without adding a bunch of PowerMockito dependencies for catching a static |
||
| } | ||
| }; | ||
| retryPolicy = new BoundedExponentialBackoffRetryWithQuit( | ||
| exitRunner, | ||
| BASE_SLEEP_TIME_MS, | ||
| MAX_SLEEP_TIME_MS, | ||
| MAX_RETRIES | ||
| ); | ||
| } else { | ||
| retryPolicy = new BoundedExponentialBackoffRetry(BASE_SLEEP_TIME_MS, MAX_SLEEP_TIME_MS, MAX_RETRIES); | ||
| } | ||
|
|
||
| final CuratorFramework framework = builder | ||
| .ensembleProvider(ensembleProvider) | ||
| .sessionTimeoutMs(config.getZkSessionTimeoutMs()) | ||
| .retryPolicy(new BoundedExponentialBackoffRetry(BASE_SLEEP_TIME_MS, MAX_SLEEP_TIME_MS, MAX_RETRIES)) | ||
| .retryPolicy(retryPolicy) | ||
| .compressionProvider(new PotentiallyGzippedCompressionProvider(config.getEnableCompression())) | ||
| .aclProvider(config.getEnableAcl() ? new SecuredACLProvider() : new DefaultACLProvider()) | ||
| .build(); | ||
|
|
@@ -127,6 +153,30 @@ public EnsembleProvider makeEnsembleProvider(CuratorConfig config, ExhibitorConf | |
| return new FixedEnsembleProvider(config.getZkHosts()); | ||
| } | ||
|
|
||
| RetryPolicy retryPolicy; | ||
| if (config.getTerminateDruidProcessOnConnectFail()) { | ||
| // It's unknown whether or not this precaution is needed. Tests revealed that this path was never taken. | ||
| // see discussions in https://github.com/apache/incubator-druid/pull/6740 | ||
|
|
||
| final Runnable exitRunner = () -> { | ||
| try { | ||
| log.error("Zookeeper can't be reached, forcefully stopping virtual machine..."); | ||
| } | ||
| finally { | ||
| System.exit(1); | ||
|
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. try {
log.error("Zookeeper can't be reached, forcefully stopping virtual machine...");
} finally {
System.exit(1);
}
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Switched to this pattern. I didn't think it was necessary since I've never seen a logger kill a JVM.
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Should this stop the lifecycle too? If this is getting hit it means that the curator killer is not getting hit and so I think this exit would be unclean. Also, did you figure out why this needs to kill the process too? I see the discussion here, #6740 (comment) but it doesn't really come to a conclusion about whether it's needed. I'm not sure it is needed. Presumably if exhibitor is down the zk processes managed by it are dead too, and I would expect the other killer will get hit? Or is this for a startup case where the other killer somehow doesn't get triggered? Regardless, could you also add a comment with the explanation?
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @clintropolis I think the problem is that nobody knows answers to your questions and nobody has the determination to invest the effort to research this question (yet). I don't think we should demand this from @michael-trelinski. I think the PR in its current form is a move in the right direction anyway. However, @michael-trelinski could you please add a comment in
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Sure, I wasn't expecting a deep investigation, I was suggesting either adding a
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Arguments to those provider methods are injected, so it could be available if you added it as an argument, e.g.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. In
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I see what you're getting at. Apologies. However, if it wasn't there to begin with and we're all not sure why makeEnsembleProvider is even there, should I be mucking around with it by adding a Lifecycle?
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I don't think there is question of why The only question I had is whether the quit on retry functionality needs to be wired into it's failures as well as curator's failures. I believe there should be no harm in using the injected lifecycle here as far as I can tell.
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @michael-trelinski could you please link this discussion from the comment? There is no superstition that code should be "self-contained" in Druid, there are many links from the code to Github. It would be easier for readers to navigate. Currently, readers would be forced to dig into Git history to follow this path. |
||
| } | ||
| }; | ||
|
|
||
| retryPolicy = new BoundedExponentialBackoffRetryWithQuit( | ||
| exitRunner, | ||
| BASE_SLEEP_TIME_MS, | ||
| MAX_SLEEP_TIME_MS, | ||
| MAX_RETRIES | ||
| ); | ||
| } else { | ||
| retryPolicy = new BoundedExponentialBackoffRetry(BASE_SLEEP_TIME_MS, MAX_SLEEP_TIME_MS, MAX_RETRIES); | ||
| } | ||
|
|
||
| return new ExhibitorEnsembleProvider( | ||
| new Exhibitors( | ||
| exConfig.getHosts(), | ||
|
|
@@ -136,7 +186,7 @@ public EnsembleProvider makeEnsembleProvider(CuratorConfig config, ExhibitorConf | |
| new DefaultExhibitorRestClient(exConfig.getUseSsl()), | ||
| exConfig.getRestUriPath(), | ||
| exConfig.getPollingMs(), | ||
| new BoundedExponentialBackoffRetry(BASE_SLEEP_TIME_MS, MAX_SLEEP_TIME_MS, MAX_RETRIES) | ||
| retryPolicy | ||
| ) | ||
| { | ||
| @Override | ||
|
|
||
| Original file line number | Diff line number | Diff line change |
|---|---|---|
| @@ -0,0 +1,111 @@ | ||
| /* | ||
| * Licensed to the Apache Software Foundation (ASF) under one | ||
| * or more contributor license agreements. See the NOTICE file | ||
| * distributed with this work for additional information | ||
| * regarding copyright ownership. The ASF licenses this file | ||
| * to you under the Apache License, Version 2.0 (the | ||
| * "License"); you may not use this file except in compliance | ||
| * with the License. You may obtain a copy of the License at | ||
| * | ||
| * http://www.apache.org/licenses/LICENSE-2.0 | ||
| * | ||
| * Unless required by applicable law or agreed to in writing, | ||
| * software distributed under the License is distributed on an | ||
| * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY | ||
| * KIND, either express or implied. See the License for the | ||
| * specific language governing permissions and limitations | ||
| * under the License. | ||
| */ | ||
|
|
||
| package org.apache.druid.curator; | ||
|
|
||
| import org.apache.curator.framework.CuratorFramework; | ||
| import org.apache.curator.framework.CuratorFrameworkFactory; | ||
| import org.apache.curator.test.TestingServer; | ||
| import org.apache.druid.java.util.common.lifecycle.Lifecycle; | ||
| import org.apache.druid.java.util.common.logger.Logger; | ||
| import org.easymock.EasyMock; | ||
| import org.junit.Assert; | ||
| import org.junit.Test; | ||
|
|
||
| public final class BoundedExponentialBackoffRetryWithQuitTest | ||
| { | ||
|
|
||
| private static final Logger log = new Logger(BoundedExponentialBackoffRetryWithQuitTest.class); | ||
|
|
||
| /* | ||
|
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. (Optional) no reason why it shouldn't be a Javadoc comment.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
@leventov I don't understand what you mean.
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I mean that comments in code should say
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @leventov Ok, done.
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @michael-trelinski this is a wrong place. This link should be a part of the comment in // It's unknown whether or not this precaution is needed. Tests revealed that this path was never taken.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Apologies. It's part of the thread-view for BoundedExponentialBackoffRetryWithQuitTest.java. |
||
| Methodology (order is important!): | ||
| 1. Zookeeper Server Service started | ||
| 2. Lifecycle started | ||
| 3. Curator invokes connection to service | ||
| 4. Service is stopped | ||
| 5. Curator attempts to do something, which invokes the retries policy | ||
| 6. Retries exceed limit, call function which simulates an exit (since mocking System.exit() is hard to do without | ||
| changing a lot of dependencies) | ||
| */ | ||
| @Test | ||
| public void testExitWithLifecycle() throws Exception | ||
| { | ||
| final Lifecycle actualNoop = new Lifecycle() { | ||
| @Override | ||
| public void start() throws Exception | ||
| { | ||
| super.start(); | ||
| log.info("Starting lifecycle..."); | ||
| } | ||
|
|
||
| @Override | ||
| public void stop() | ||
| { | ||
| super.stop(); | ||
| log.info("Stopping lifecycle..."); | ||
| } | ||
| }; | ||
| Lifecycle noop = EasyMock.mock(Lifecycle.class); | ||
|
|
||
| noop.start(); | ||
| EasyMock.expectLastCall().andDelegateTo(actualNoop); | ||
| noop.stop(); | ||
| EasyMock.expectLastCall().andDelegateTo(actualNoop); | ||
| EasyMock.replay(noop); | ||
|
|
||
| Runnable exitFunction = () -> { | ||
| log.info("Zookeeper retries exhausted, exiting..."); | ||
| noop.stop(); | ||
| throw new RuntimeException("Simulated exit"); | ||
| }; | ||
|
|
||
| TestingServer server = new TestingServer(); | ||
| BoundedExponentialBackoffRetryWithQuit retry = new BoundedExponentialBackoffRetryWithQuit(exitFunction, 1, 1, 2); | ||
| CuratorFramework curator = CuratorFrameworkFactory | ||
| .builder() | ||
| .connectString(server.getConnectString()) | ||
| .sessionTimeoutMs(1000) | ||
| .connectionTimeoutMs(1) | ||
| .retryPolicy(retry) | ||
| .build(); | ||
| server.start(); | ||
| System.out.println("Server started."); | ||
| curator.start(); | ||
| noop.start(); | ||
| curator.checkExists().forPath("/tmp"); | ||
| log.info("Connected."); | ||
| boolean failed = false; | ||
| try { | ||
| server.stop(); | ||
| log.info("Stopped."); | ||
| curator.checkExists().forPath("/tmp"); | ||
| Thread.sleep(10); | ||
| curator.checkExists().forPath("/tmp"); | ||
| } | ||
| catch (Exception e) { | ||
| Assert.assertTrue("Correct exception type", e instanceof RuntimeException); | ||
| EasyMock.verify(noop); | ||
| curator.close(); | ||
| failed = true; | ||
| } | ||
| Assert.assertTrue("Must be marked in failure state", failed); | ||
| log.info("Lifecycle stopped."); | ||
| } | ||
|
|
||
| } | ||
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.
Maybe use composition instead of inheritance?
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.
Point taken, but I think this is simple, and if the base class changes its constructor, we'll have to refactor either way. I wanted this based off of the BoundedExponentialBackoffRetry logic, so I subclassed it.
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.
Could you please add a Javadoc comment "BoundedExponentialBackoffRetryWithQuit extends BoundedExponentialBackoffRetry for simplicity. It's not actually a BoundedExponentialBackoffRetry from the Liskov substitution principle point of view, but it doesn't matter in this code."