diff --git a/server/src/main/java/org/elasticsearch/cluster/service/MasterService.java b/server/src/main/java/org/elasticsearch/cluster/service/MasterService.java index 20a6602b5c5ad..54a6568af3fa2 100644 --- a/server/src/main/java/org/elasticsearch/cluster/service/MasterService.java +++ b/server/src/main/java/org/elasticsearch/cluster/service/MasterService.java @@ -386,7 +386,7 @@ public Discovery.AckListener createAckListener(ThreadPool threadPool, ClusterSta } }); - return new DelegetingAckListener(ackListeners); + return new DelegatingAckListener(ackListeners); } public boolean clusterStateUnchanged() { @@ -541,11 +541,11 @@ protected void warnAboutSlowTaskIfNeeded(TimeValue executionTime, String source) } } - private static class DelegetingAckListener implements Discovery.AckListener { + private static class DelegatingAckListener implements Discovery.AckListener { private final List listeners; - private DelegetingAckListener(List listeners) { + private DelegatingAckListener(List listeners) { this.listeners = listeners; } @@ -555,11 +555,6 @@ public void onNodeAck(DiscoveryNode node, @Nullable Exception e) { listener.onNodeAck(node, e); } } - - @Override - public void onTimeout() { - throw new UnsupportedOperationException("no timeout delegation"); - } } private static class AckCountDownListener implements Discovery.AckListener { @@ -614,7 +609,6 @@ public void onNodeAck(DiscoveryNode node, @Nullable Exception e) { } } - @Override public void onTimeout() { if (countDown.fastForward()) { logger.trace("timeout waiting for acknowledgement for cluster_state update (version: {})", clusterStateVersion); diff --git a/server/src/main/java/org/elasticsearch/discovery/Discovery.java b/server/src/main/java/org/elasticsearch/discovery/Discovery.java index 3842e68d1006b..9c70876032442 100644 --- a/server/src/main/java/org/elasticsearch/discovery/Discovery.java +++ b/server/src/main/java/org/elasticsearch/discovery/Discovery.java @@ -49,7 +49,6 @@ public interface Discovery extends LifecycleComponent { interface AckListener { void onNodeAck(DiscoveryNode node, @Nullable Exception e); - void onTimeout(); } class FailedToCommitClusterStateException extends ElasticsearchException { diff --git a/server/src/test/java/org/elasticsearch/discovery/zen/PublishClusterStateActionTests.java b/server/src/test/java/org/elasticsearch/discovery/zen/PublishClusterStateActionTests.java index 42ec72c981007..c8e85382994c7 100644 --- a/server/src/test/java/org/elasticsearch/discovery/zen/PublishClusterStateActionTests.java +++ b/server/src/test/java/org/elasticsearch/discovery/zen/PublishClusterStateActionTests.java @@ -814,7 +814,6 @@ public AssertingAckListener publishState(PublishClusterStateAction action, Clust public static class AssertingAckListener implements Discovery.AckListener { private final List> errors = new CopyOnWriteArrayList<>(); - private final AtomicBoolean timeoutOccurred = new AtomicBoolean(); private final CountDownLatch countDown; public AssertingAckListener(int nodeCount) { @@ -829,23 +828,12 @@ public void onNodeAck(DiscoveryNode node, @Nullable Exception e) { countDown.countDown(); } - @Override - public void onTimeout() { - timeoutOccurred.set(true); - // Fast forward the counter - no reason to wait here - long currentCount = countDown.getCount(); - for (long i = 0; i < currentCount; i++) { - countDown.countDown(); - } - } - public void await(long timeout, TimeUnit unit) throws InterruptedException { assertThat(awaitErrors(timeout, unit), emptyIterable()); } public List> awaitErrors(long timeout, TimeUnit unit) throws InterruptedException { countDown.await(timeout, unit); - assertFalse(timeoutOccurred.get()); return errors; }