|
@@ -15,14 +15,14 @@ import org.elasticsearch.action.admin.cluster.state.ClusterStateRequest;
|
|
|
import org.elasticsearch.client.Client;
|
|
|
import org.elasticsearch.cluster.ClusterChangedEvent;
|
|
|
import org.elasticsearch.cluster.ClusterState;
|
|
|
-import org.elasticsearch.cluster.ClusterStateApplier;
|
|
|
+import org.elasticsearch.cluster.ClusterStateListener;
|
|
|
import org.elasticsearch.cluster.ClusterStateUpdateTask;
|
|
|
import org.elasticsearch.cluster.metadata.IndexMetaData;
|
|
|
import org.elasticsearch.cluster.metadata.MetaData;
|
|
|
import org.elasticsearch.cluster.routing.IndexRoutingTable;
|
|
|
import org.elasticsearch.cluster.service.ClusterService;
|
|
|
+import org.elasticsearch.common.collect.CopyOnWriteHashMap;
|
|
|
import org.elasticsearch.common.collect.Tuple;
|
|
|
-import org.elasticsearch.common.settings.Settings;
|
|
|
import org.elasticsearch.common.unit.TimeValue;
|
|
|
import org.elasticsearch.common.util.concurrent.AtomicArray;
|
|
|
import org.elasticsearch.common.util.concurrent.CountDown;
|
|
@@ -31,7 +31,6 @@ import org.elasticsearch.index.IndexSettings;
|
|
|
import org.elasticsearch.license.LicenseUtils;
|
|
|
import org.elasticsearch.threadpool.ThreadPool;
|
|
|
import org.elasticsearch.xpack.ccr.CcrLicenseChecker;
|
|
|
-import org.elasticsearch.xpack.ccr.CcrSettings;
|
|
|
import org.elasticsearch.xpack.core.ccr.AutoFollowMetadata;
|
|
|
import org.elasticsearch.xpack.core.ccr.AutoFollowMetadata.AutoFollowPattern;
|
|
|
import org.elasticsearch.xpack.core.ccr.AutoFollowStats;
|
|
@@ -45,28 +44,29 @@ import java.util.LinkedHashMap;
|
|
|
import java.util.List;
|
|
|
import java.util.Map;
|
|
|
import java.util.Objects;
|
|
|
+import java.util.Set;
|
|
|
import java.util.TreeMap;
|
|
|
import java.util.function.BiConsumer;
|
|
|
import java.util.function.Consumer;
|
|
|
import java.util.function.Function;
|
|
|
+import java.util.function.Supplier;
|
|
|
import java.util.stream.Collectors;
|
|
|
|
|
|
/**
|
|
|
* A component that runs only on the elected master node and follows leader indices automatically
|
|
|
* if they match with a auto follow pattern that is defined in {@link AutoFollowMetadata}.
|
|
|
*/
|
|
|
-public class AutoFollowCoordinator implements ClusterStateApplier {
|
|
|
+public class AutoFollowCoordinator implements ClusterStateListener {
|
|
|
|
|
|
private static final Logger LOGGER = LogManager.getLogger(AutoFollowCoordinator.class);
|
|
|
private static final int MAX_AUTO_FOLLOW_ERRORS = 256;
|
|
|
|
|
|
private final Client client;
|
|
|
- private final TimeValue pollInterval;
|
|
|
private final ThreadPool threadPool;
|
|
|
private final ClusterService clusterService;
|
|
|
private final CcrLicenseChecker ccrLicenseChecker;
|
|
|
|
|
|
- private volatile boolean localNodeMaster = false;
|
|
|
+ private volatile Map<String, AutoFollower> autoFollowers = Collections.emptyMap();
|
|
|
|
|
|
// The following fields are read and updated under a lock:
|
|
|
private long numberOfSuccessfulIndicesAutoFollowed = 0;
|
|
@@ -75,7 +75,6 @@ public class AutoFollowCoordinator implements ClusterStateApplier {
|
|
|
private final LinkedHashMap<String, ElasticsearchException> recentAutoFollowErrors;
|
|
|
|
|
|
public AutoFollowCoordinator(
|
|
|
- Settings settings,
|
|
|
Client client,
|
|
|
ThreadPool threadPool,
|
|
|
ClusterService clusterService,
|
|
@@ -84,10 +83,7 @@ public class AutoFollowCoordinator implements ClusterStateApplier {
|
|
|
this.threadPool = threadPool;
|
|
|
this.clusterService = clusterService;
|
|
|
this.ccrLicenseChecker = Objects.requireNonNull(ccrLicenseChecker, "ccrLicenseChecker");
|
|
|
-
|
|
|
- this.pollInterval = CcrSettings.CCR_AUTO_FOLLOW_POLL_INTERVAL.get(settings);
|
|
|
- clusterService.addStateApplier(this);
|
|
|
-
|
|
|
+ clusterService.addListener(this);
|
|
|
this.recentAutoFollowErrors = new LinkedHashMap<String, ElasticsearchException>() {
|
|
|
@Override
|
|
|
protected boolean removeEldestEntry(final Map.Entry<String, ElasticsearchException> eldest) {
|
|
@@ -130,151 +126,189 @@ public class AutoFollowCoordinator implements ClusterStateApplier {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- private void doAutoFollow() {
|
|
|
- if (localNodeMaster == false) {
|
|
|
- return;
|
|
|
- }
|
|
|
- ClusterState followerClusterState = clusterService.state();
|
|
|
+ void updateAutoFollowers(ClusterState followerClusterState) {
|
|
|
AutoFollowMetadata autoFollowMetadata = followerClusterState.getMetaData().custom(AutoFollowMetadata.TYPE);
|
|
|
if (autoFollowMetadata == null) {
|
|
|
- threadPool.schedule(pollInterval, ThreadPool.Names.SAME, this::doAutoFollow);
|
|
|
- return;
|
|
|
- }
|
|
|
-
|
|
|
- if (autoFollowMetadata.getPatterns().isEmpty()) {
|
|
|
- threadPool.schedule(pollInterval, ThreadPool.Names.SAME, this::doAutoFollow);
|
|
|
return;
|
|
|
}
|
|
|
|
|
|
if (ccrLicenseChecker.isCcrAllowed() == false) {
|
|
|
// TODO: set non-compliant status on auto-follow coordination that can be viewed via a stats API
|
|
|
LOGGER.warn("skipping auto-follower coordination", LicenseUtils.newComplianceException("ccr"));
|
|
|
- threadPool.schedule(pollInterval, ThreadPool.Names.SAME, this::doAutoFollow);
|
|
|
return;
|
|
|
}
|
|
|
|
|
|
- Consumer<List<AutoFollowResult>> handler = results -> {
|
|
|
- updateStats(results);
|
|
|
- threadPool.schedule(pollInterval, ThreadPool.Names.SAME, this::doAutoFollow);
|
|
|
- };
|
|
|
- AutoFollower operation = new AutoFollower(handler, followerClusterState) {
|
|
|
+ final CopyOnWriteHashMap<String, AutoFollower> autoFollowers = CopyOnWriteHashMap.copyOf(this.autoFollowers);
|
|
|
+ Set<String> newRemoteClusters = autoFollowMetadata.getPatterns().entrySet().stream()
|
|
|
+ .map(entry -> entry.getValue().getRemoteCluster())
|
|
|
+ .filter(remoteCluster -> autoFollowers.containsKey(remoteCluster) == false)
|
|
|
+ .collect(Collectors.toSet());
|
|
|
+
|
|
|
+ Map<String, AutoFollower> newAutoFollowers = new HashMap<>(newRemoteClusters.size());
|
|
|
+ for (String remoteCluster : newRemoteClusters) {
|
|
|
+ AutoFollower autoFollower = new AutoFollower(remoteCluster, threadPool, this::updateStats, clusterService::state) {
|
|
|
+
|
|
|
+ @Override
|
|
|
+ void getLeaderClusterState(final String remoteCluster,
|
|
|
+ final BiConsumer<ClusterState, Exception> handler) {
|
|
|
+ final ClusterStateRequest request = new ClusterStateRequest();
|
|
|
+ request.clear();
|
|
|
+ request.metaData(true);
|
|
|
+ request.routingTable(true);
|
|
|
+ // TODO: set non-compliant status on auto-follow coordination that can be viewed via a stats API
|
|
|
+ ccrLicenseChecker.checkRemoteClusterLicenseAndFetchClusterState(
|
|
|
+ client,
|
|
|
+ remoteCluster,
|
|
|
+ request,
|
|
|
+ e -> handler.accept(null, e),
|
|
|
+ leaderClusterState -> handler.accept(leaderClusterState, null));
|
|
|
+ }
|
|
|
|
|
|
- @Override
|
|
|
- void getLeaderClusterState(final String remoteCluster,
|
|
|
- final BiConsumer<ClusterState, Exception> handler) {
|
|
|
- final ClusterStateRequest request = new ClusterStateRequest();
|
|
|
- request.clear();
|
|
|
- request.metaData(true);
|
|
|
- request.routingTable(true);
|
|
|
- // TODO: set non-compliant status on auto-follow coordination that can be viewed via a stats API
|
|
|
- ccrLicenseChecker.checkRemoteClusterLicenseAndFetchClusterState(
|
|
|
- client,
|
|
|
- remoteCluster,
|
|
|
- request,
|
|
|
- e -> handler.accept(null, e),
|
|
|
- leaderClusterState -> handler.accept(leaderClusterState, null));
|
|
|
- }
|
|
|
+ @Override
|
|
|
+ void createAndFollow(Map<String, String> headers,
|
|
|
+ PutFollowAction.Request request,
|
|
|
+ Runnable successHandler,
|
|
|
+ Consumer<Exception> failureHandler) {
|
|
|
+ Client followerClient = CcrLicenseChecker.wrapClient(client, headers);
|
|
|
+ followerClient.execute(
|
|
|
+ PutFollowAction.INSTANCE,
|
|
|
+ request,
|
|
|
+ ActionListener.wrap(r -> successHandler.run(), failureHandler)
|
|
|
+ );
|
|
|
+ }
|
|
|
|
|
|
- @Override
|
|
|
- void createAndFollow(Map<String, String> headers,
|
|
|
- PutFollowAction.Request request,
|
|
|
- Runnable successHandler,
|
|
|
- Consumer<Exception> failureHandler) {
|
|
|
- Client followerClient = CcrLicenseChecker.wrapClient(client, headers);
|
|
|
- followerClient.execute(
|
|
|
- PutFollowAction.INSTANCE,
|
|
|
- request,
|
|
|
- ActionListener.wrap(r -> successHandler.run(), failureHandler)
|
|
|
- );
|
|
|
- }
|
|
|
+ @Override
|
|
|
+ void updateAutoFollowMetadata(Function<ClusterState, ClusterState> updateFunction,
|
|
|
+ Consumer<Exception> handler) {
|
|
|
+ clusterService.submitStateUpdateTask("update_auto_follow_metadata", new ClusterStateUpdateTask() {
|
|
|
|
|
|
- @Override
|
|
|
- void updateAutoFollowMetadata(Function<ClusterState, ClusterState> updateFunction,
|
|
|
- Consumer<Exception> handler) {
|
|
|
- clusterService.submitStateUpdateTask("update_auto_follow_metadata", new ClusterStateUpdateTask() {
|
|
|
+ @Override
|
|
|
+ public ClusterState execute(ClusterState currentState) throws Exception {
|
|
|
+ return updateFunction.apply(currentState);
|
|
|
+ }
|
|
|
|
|
|
- @Override
|
|
|
- public ClusterState execute(ClusterState currentState) throws Exception {
|
|
|
- return updateFunction.apply(currentState);
|
|
|
- }
|
|
|
+ @Override
|
|
|
+ public void onFailure(String source, Exception e) {
|
|
|
+ handler.accept(e);
|
|
|
+ }
|
|
|
|
|
|
- @Override
|
|
|
- public void onFailure(String source, Exception e) {
|
|
|
- handler.accept(e);
|
|
|
- }
|
|
|
+ @Override
|
|
|
+ public void clusterStateProcessed(String source, ClusterState oldState, ClusterState newState) {
|
|
|
+ handler.accept(null);
|
|
|
+ }
|
|
|
+ });
|
|
|
+ }
|
|
|
|
|
|
- @Override
|
|
|
- public void clusterStateProcessed(String source, ClusterState oldState, ClusterState newState) {
|
|
|
- handler.accept(null);
|
|
|
- }
|
|
|
- });
|
|
|
- }
|
|
|
+ };
|
|
|
+ newAutoFollowers.put(remoteCluster, autoFollower);
|
|
|
+ autoFollower.autoFollowIndices();
|
|
|
+ }
|
|
|
|
|
|
- };
|
|
|
- operation.autoFollowIndices();
|
|
|
+ List<String> removedRemoteClusters = new ArrayList<>();
|
|
|
+ for (String remoteCluster : autoFollowers.keySet()) {
|
|
|
+ boolean exist = autoFollowMetadata.getPatterns().values().stream()
|
|
|
+ .anyMatch(pattern -> pattern.getRemoteCluster().equals(remoteCluster));
|
|
|
+ if (exist == false) {
|
|
|
+ removedRemoteClusters.add(remoteCluster);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ this.autoFollowers = autoFollowers
|
|
|
+ .copyAndPutAll(newAutoFollowers)
|
|
|
+ .copyAndRemoveAll(removedRemoteClusters);
|
|
|
}
|
|
|
|
|
|
@Override
|
|
|
- public void applyClusterState(ClusterChangedEvent event) {
|
|
|
- final boolean beforeLocalMasterNode = localNodeMaster;
|
|
|
- localNodeMaster = event.localNodeMaster();
|
|
|
- if (beforeLocalMasterNode == false && localNodeMaster) {
|
|
|
- threadPool.schedule(pollInterval, ThreadPool.Names.SAME, this::doAutoFollow);
|
|
|
+ public void clusterChanged(ClusterChangedEvent event) {
|
|
|
+ if (event.localNodeMaster()) {
|
|
|
+ updateAutoFollowers(event.state());
|
|
|
}
|
|
|
}
|
|
|
|
|
|
+ /**
|
|
|
+ * Each auto follower independently monitors a remote cluster for new leader indices that should be auto followed.
|
|
|
+ * The reason that this should happen independently, is that when auto followers start to make use of cluster state
|
|
|
+ * API's 'wait_for_metadata_version' feature, it may take sometime before a remote cluster responds with a new
|
|
|
+ * cluster state or times out. Other auto follow patterns for different remote clusters are then forced to wait,
|
|
|
+ * which can cause new follower indices to unnecessarily start with a large backlog of operations that need to be
|
|
|
+ * replicated.
|
|
|
+ */
|
|
|
abstract static class AutoFollower {
|
|
|
|
|
|
- private final Consumer<List<AutoFollowResult>> handler;
|
|
|
- private final ClusterState followerClusterState;
|
|
|
- private final AutoFollowMetadata autoFollowMetadata;
|
|
|
-
|
|
|
- private final CountDown autoFollowPatternsCountDown;
|
|
|
- private final AtomicArray<AutoFollowResult> autoFollowResults;
|
|
|
-
|
|
|
- AutoFollower(final Consumer<List<AutoFollowResult>> handler, final ClusterState followerClusterState) {
|
|
|
- this.handler = handler;
|
|
|
- this.followerClusterState = followerClusterState;
|
|
|
- this.autoFollowMetadata = followerClusterState.getMetaData().custom(AutoFollowMetadata.TYPE);
|
|
|
- this.autoFollowPatternsCountDown = new CountDown(autoFollowMetadata.getPatterns().size());
|
|
|
- this.autoFollowResults = new AtomicArray<>(autoFollowMetadata.getPatterns().size());
|
|
|
+ private final String remoteCluster;
|
|
|
+ private final ThreadPool threadPool;
|
|
|
+ private final Consumer<List<AutoFollowResult>> statsUpdater;
|
|
|
+ private final Supplier<ClusterState> followerClusterStateSupplier;
|
|
|
+
|
|
|
+ private volatile CountDown autoFollowPatternsCountDown;
|
|
|
+ private volatile AtomicArray<AutoFollowResult> autoFollowResults;
|
|
|
+
|
|
|
+ AutoFollower(final String remoteCluster,
|
|
|
+ final ThreadPool threadPool,
|
|
|
+ final Consumer<List<AutoFollowResult>> statsUpdater,
|
|
|
+ final Supplier<ClusterState> followerClusterStateSupplier) {
|
|
|
+ this.remoteCluster = remoteCluster;
|
|
|
+ this.threadPool = threadPool;
|
|
|
+ this.statsUpdater = statsUpdater;
|
|
|
+ this.followerClusterStateSupplier = followerClusterStateSupplier;
|
|
|
}
|
|
|
|
|
|
void autoFollowIndices() {
|
|
|
- int i = 0;
|
|
|
- for (Map.Entry<String, AutoFollowPattern> entry : autoFollowMetadata.getPatterns().entrySet()) {
|
|
|
- final int slot = i;
|
|
|
- final String autoFollowPattenName = entry.getKey();
|
|
|
- final AutoFollowPattern autoFollowPattern = entry.getValue();
|
|
|
- final String remoteCluster = autoFollowPattern.getRemoteCluster();
|
|
|
-
|
|
|
- Map<String, String> headers = autoFollowMetadata.getHeaders().get(autoFollowPattenName);
|
|
|
- getLeaderClusterState(remoteCluster, (leaderClusterState, e) -> {
|
|
|
- if (leaderClusterState != null) {
|
|
|
- assert e == null;
|
|
|
- final List<String> followedIndices = autoFollowMetadata.getFollowedLeaderIndexUUIDs().get(autoFollowPattenName);
|
|
|
- final List<Index> leaderIndicesToFollow = getLeaderIndicesToFollow(remoteCluster, autoFollowPattern,
|
|
|
- leaderClusterState, followerClusterState, followedIndices);
|
|
|
+ final ClusterState followerClusterState = followerClusterStateSupplier.get();
|
|
|
+ final AutoFollowMetadata autoFollowMetadata = followerClusterState.metaData().custom(AutoFollowMetadata.TYPE);
|
|
|
+ if (autoFollowMetadata == null) {
|
|
|
+ LOGGER.info("AutoFollower for cluster [{}] has stopped, because there is no autofollow metadata", remoteCluster);
|
|
|
+ return;
|
|
|
+ }
|
|
|
+
|
|
|
+ final List<String> patterns = autoFollowMetadata.getPatterns().entrySet().stream()
|
|
|
+ .filter(entry -> entry.getValue().getRemoteCluster().equals(remoteCluster))
|
|
|
+ .map(Map.Entry::getKey)
|
|
|
+ .collect(Collectors.toList());
|
|
|
+ if (patterns.isEmpty()) {
|
|
|
+ LOGGER.info("AutoFollower for cluster [{}] has stopped, because there are no more patterns", remoteCluster);
|
|
|
+ return;
|
|
|
+ }
|
|
|
+
|
|
|
+ this.autoFollowPatternsCountDown = new CountDown(patterns.size());
|
|
|
+ this.autoFollowResults = new AtomicArray<>(patterns.size());
|
|
|
+
|
|
|
+ getLeaderClusterState(remoteCluster, (leaderClusterState, e) -> {
|
|
|
+ if (leaderClusterState != null) {
|
|
|
+ assert e == null;
|
|
|
+
|
|
|
+ int i = 0;
|
|
|
+ for (String autoFollowPatternName : patterns) {
|
|
|
+ final int slot = i;
|
|
|
+ AutoFollowPattern autoFollowPattern = autoFollowMetadata.getPatterns().get(autoFollowPatternName);
|
|
|
+ Map<String, String> headers = autoFollowMetadata.getHeaders().get(autoFollowPatternName);
|
|
|
+ List<String> followedIndices = autoFollowMetadata.getFollowedLeaderIndexUUIDs().get(autoFollowPatternName);
|
|
|
+
|
|
|
+ final List<Index> leaderIndicesToFollow = getLeaderIndicesToFollow(autoFollowPattern, leaderClusterState,
|
|
|
+ followerClusterState, followedIndices);
|
|
|
if (leaderIndicesToFollow.isEmpty()) {
|
|
|
- finalise(slot, new AutoFollowResult(autoFollowPattenName));
|
|
|
+ finalise(slot, new AutoFollowResult(autoFollowPatternName));
|
|
|
} else {
|
|
|
List<Tuple<String, AutoFollowPattern>> patternsForTheSameLeaderCluster = autoFollowMetadata.getPatterns()
|
|
|
.entrySet().stream()
|
|
|
- .filter(item -> autoFollowPattenName.equals(item.getKey()) == false)
|
|
|
+ .filter(item -> autoFollowPatternName.equals(item.getKey()) == false)
|
|
|
.filter(item -> remoteCluster.equals(item.getValue().getRemoteCluster()))
|
|
|
.map(item -> new Tuple<>(item.getKey(), item.getValue()))
|
|
|
.collect(Collectors.toList());
|
|
|
|
|
|
Consumer<AutoFollowResult> resultHandler = result -> finalise(slot, result);
|
|
|
- checkAutoFollowPattern(autoFollowPattenName, remoteCluster, autoFollowPattern, leaderIndicesToFollow, headers,
|
|
|
+ checkAutoFollowPattern(autoFollowPatternName, remoteCluster, autoFollowPattern, leaderIndicesToFollow, headers,
|
|
|
patternsForTheSameLeaderCluster, resultHandler);
|
|
|
}
|
|
|
- } else {
|
|
|
- finalise(slot, new AutoFollowResult(autoFollowPattenName, e));
|
|
|
+ i++;
|
|
|
}
|
|
|
- });
|
|
|
- i++;
|
|
|
- }
|
|
|
+ } else {
|
|
|
+ List<AutoFollowResult> results = new ArrayList<>(patterns.size());
|
|
|
+ for (String autoFollowPatternName : patterns) {
|
|
|
+ results.add(new AutoFollowResult(autoFollowPatternName, e));
|
|
|
+ }
|
|
|
+ statsUpdater.accept(results);
|
|
|
+ }
|
|
|
+ });
|
|
|
}
|
|
|
|
|
|
private void checkAutoFollowPattern(String autoFollowPattenName,
|
|
@@ -357,12 +391,13 @@ public class AutoFollowCoordinator implements ClusterStateApplier {
|
|
|
assert autoFollowResults.get(slot) == null;
|
|
|
autoFollowResults.set(slot, result);
|
|
|
if (autoFollowPatternsCountDown.countDown()) {
|
|
|
- handler.accept(autoFollowResults.asList());
|
|
|
+ statsUpdater.accept(autoFollowResults.asList());
|
|
|
+ // TODO: Remove scheduling here with using cluster state API's waitForMetadataVersion:
|
|
|
+ threadPool.schedule(TimeValue.timeValueMillis(2500), ThreadPool.Names.GENERIC, this::autoFollowIndices);
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- static List<Index> getLeaderIndicesToFollow(String remoteCluster,
|
|
|
- AutoFollowPattern autoFollowPattern,
|
|
|
+ static List<Index> getLeaderIndicesToFollow(AutoFollowPattern autoFollowPattern,
|
|
|
ClusterState leaderClusterState,
|
|
|
ClusterState followerClusterState,
|
|
|
List<String> followedIndexUUIDs) {
|