Voting config exclusions should work with absent nodes (#55291)

Today the voting config exclusions API accepts node filters and resolves them
to a collection of node IDs against the current cluster membership.

This is problematic since we may want to exclude nodes that are not currently
members of the cluster. For instance:

- if attempting to remove a flaky node from the cluster you cannot reliably
  exclude it from the voting configuration since it may not reliably be a
  member of the cluster

- if `cluster.auto_shrink_voting_configuration: false` then naively shrinking
  the cluster will remove some nodes but will leaving their node IDs in the
  voting configuration. The only way to clean up the voting configuration is to
  grow the cluster back to its original size (potentially replacing some of the
  voting configuration) and then use the exclusions API.

This commit adds an alternative API that accepts node names and node IDs but
not node filters in general, and deprecates the current node-filters-based API.

Relates #47990.
Backport of #50836 to 7.x.

Co-authored-by: zacharymorn <zacharymorn@gmail.com>
This commit is contained in:
David Turner 2020-04-16 12:28:50 +01:00 committed by GitHub
parent ac87c10039
commit 8a565c4fa6
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
15 changed files with 684 additions and 82 deletions

View File

@ -18,70 +18,142 @@
*/
package org.elasticsearch.action.admin.cluster.configuration;
import org.apache.logging.log4j.LogManager;
import org.elasticsearch.Version;
import org.elasticsearch.action.ActionRequestValidationException;
import org.elasticsearch.action.support.master.MasterNodeRequest;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.coordination.CoordinationMetadata.VotingConfigExclusion;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.unit.TimeValue;
import org.elasticsearch.common.logging.DeprecationLogger;
import java.io.IOException;
import java.util.Arrays;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
import java.util.function.Function;
import java.util.stream.Collectors;
import java.util.stream.StreamSupport;
/**
* A request to add voting config exclusions for certain master-eligible nodes, and wait for these nodes to be removed from the voting
* configuration.
*/
public class AddVotingConfigExclusionsRequest extends MasterNodeRequest<AddVotingConfigExclusionsRequest> {
public static final String DEPRECATION_MESSAGE = "nodeDescription is deprecated and will be removed, use nodeIds or nodeNames instead";
private static final DeprecationLogger deprecationLogger = new DeprecationLogger(
LogManager.getLogger(AddVotingConfigExclusionsRequest.class));
private final String[] nodeDescriptions;
private final String[] nodeIds;
private final String[] nodeNames;
private final TimeValue timeout;
/**
* Construct a request to add voting config exclusions for master-eligible nodes matching the given descriptions, and wait for a
* Construct a request to add voting config exclusions for master-eligible nodes matching the given node names, and wait for a
* default 30 seconds for these exclusions to take effect, removing the nodes from the voting configuration.
* @param nodeDescriptions Descriptions of the nodes to add - see {@link DiscoveryNodes#resolveNodes(String...)}
* @param nodeNames Names of the nodes to add - see {@link AddVotingConfigExclusionsRequest#resolveVotingConfigExclusions(ClusterState)}
*/
public AddVotingConfigExclusionsRequest(String[] nodeDescriptions) {
this(nodeDescriptions, TimeValue.timeValueSeconds(30));
public AddVotingConfigExclusionsRequest(String... nodeNames) {
this(Strings.EMPTY_ARRAY, Strings.EMPTY_ARRAY, nodeNames, TimeValue.timeValueSeconds(30));
}
/**
* Construct a request to add voting config exclusions for master-eligible nodes matching the given descriptions, and wait for these
* nodes to be removed from the voting configuration.
* @param nodeDescriptions Descriptions of the nodes whose exclusions to add - see {@link DiscoveryNodes#resolveNodes(String...)}.
* @param nodeIds Ids of the nodes whose exclusions to add - see
* {@link AddVotingConfigExclusionsRequest#resolveVotingConfigExclusions(ClusterState)}.
* @param nodeNames Names of the nodes whose exclusions to add - see
* {@link AddVotingConfigExclusionsRequest#resolveVotingConfigExclusions(ClusterState)}.
* @param timeout How long to wait for the added exclusions to take effect and be removed from the voting configuration.
*/
public AddVotingConfigExclusionsRequest(String[] nodeDescriptions, TimeValue timeout) {
public AddVotingConfigExclusionsRequest(String[] nodeDescriptions, String[] nodeIds, String[] nodeNames, TimeValue timeout) {
if (timeout.compareTo(TimeValue.ZERO) < 0) {
throw new IllegalArgumentException("timeout [" + timeout + "] must be non-negative");
}
if (noneOrMoreThanOneIsSet(nodeDescriptions, nodeIds, nodeNames)) {
throw new IllegalArgumentException("Please set node identifiers correctly. " +
"One and only one of [node_name], [node_names] and [node_ids] has to be set");
}
if (nodeDescriptions.length > 0) {
deprecationLogger.deprecatedAndMaybeLog("voting_config_exclusion", DEPRECATION_MESSAGE);
}
this.nodeDescriptions = nodeDescriptions;
this.nodeIds = nodeIds;
this.nodeNames = nodeNames;
this.timeout = timeout;
}
public AddVotingConfigExclusionsRequest(StreamInput in) throws IOException {
super(in);
nodeDescriptions = in.readStringArray();
if (in.getVersion().onOrAfter(Version.V_7_8_0)) {
nodeIds = in.readStringArray();
nodeNames = in.readStringArray();
} else {
nodeIds = Strings.EMPTY_ARRAY;
nodeNames = Strings.EMPTY_ARRAY;
}
timeout = in.readTimeValue();
if (nodeDescriptions.length > 0) {
deprecationLogger.deprecatedAndMaybeLog("voting_config_exclusion",
"nodeDescription is deprecated and will be removed, use nodeIds or nodeNames instead");
}
}
Set<VotingConfigExclusion> resolveVotingConfigExclusions(ClusterState currentState) {
final DiscoveryNodes allNodes = currentState.nodes();
final Set<VotingConfigExclusion> resolvedNodes = Arrays.stream(allNodes.resolveNodes(nodeDescriptions))
.map(allNodes::get).filter(DiscoveryNode::isMasterNode).map(VotingConfigExclusion::new).collect(Collectors.toSet());
Set<VotingConfigExclusion> newVotingConfigExclusions = new HashSet<>();
if (resolvedNodes.isEmpty()) {
throw new IllegalArgumentException("add voting config exclusions request for " + Arrays.asList(nodeDescriptions)
+ " matched no master-eligible nodes");
if (nodeDescriptions.length >= 1) {
newVotingConfigExclusions = Arrays.stream(allNodes.resolveNodes(nodeDescriptions)).map(allNodes::get)
.filter(DiscoveryNode::isMasterNode).map(VotingConfigExclusion::new).collect(Collectors.toSet());
if (newVotingConfigExclusions.isEmpty()) {
throw new IllegalArgumentException("add voting config exclusions request for " + Arrays.asList(nodeDescriptions)
+ " matched no master-eligible nodes");
}
} else if (nodeIds.length >= 1) {
for (String nodeId : nodeIds) {
if (allNodes.nodeExists(nodeId)) {
DiscoveryNode discoveryNode = allNodes.get(nodeId);
if (discoveryNode.isMasterNode()) {
newVotingConfigExclusions.add(new VotingConfigExclusion(discoveryNode));
}
} else {
newVotingConfigExclusions.add(new VotingConfigExclusion(nodeId, VotingConfigExclusion.MISSING_VALUE_MARKER));
}
}
} else {
assert nodeNames.length >= 1;
Map<String, DiscoveryNode> existingNodes = StreamSupport.stream(allNodes.spliterator(), false)
.collect(Collectors.toMap(DiscoveryNode::getName, Function.identity()));
for (String nodeName : nodeNames) {
if (existingNodes.containsKey(nodeName)){
DiscoveryNode discoveryNode = existingNodes.get(nodeName);
if (discoveryNode.isMasterNode()) {
newVotingConfigExclusions.add(new VotingConfigExclusion(discoveryNode));
}
} else {
newVotingConfigExclusions.add(new VotingConfigExclusion(VotingConfigExclusion.MISSING_VALUE_MARKER, nodeName));
}
}
}
resolvedNodes.removeIf(n -> currentState.getVotingConfigExclusions().contains(n));
return resolvedNodes;
newVotingConfigExclusions.removeIf(n -> currentState.getVotingConfigExclusions().contains(n));
return newVotingConfigExclusions;
}
Set<VotingConfigExclusion> resolveVotingConfigExclusionsAndCheckMaximum(ClusterState currentState, int maxExclusionsCount,
@ -99,6 +171,16 @@ public class AddVotingConfigExclusionsRequest extends MasterNodeRequest<AddVotin
return resolvedExclusions;
}
private boolean noneOrMoreThanOneIsSet(String[] deprecatedNodeDescription, String[] nodeIds, String[] nodeNames) {
if (deprecatedNodeDescription.length > 0) {
return nodeIds.length > 0 || nodeNames.length > 0;
} else if (nodeIds.length > 0) {
return nodeNames.length > 0;
} else {
return nodeNames.length > 0 == false;
}
}
/**
* @return descriptions of the nodes for whom to add voting config exclusions.
*/
@ -106,6 +188,20 @@ public class AddVotingConfigExclusionsRequest extends MasterNodeRequest<AddVotin
return nodeDescriptions;
}
/**
* @return ids of the nodes for whom to add voting config exclusions.
*/
public String[] getNodeIds() {
return nodeIds;
}
/**
* @return names of the nodes for whom to add voting config exclusions.
*/
public String[] getNodeNames() {
return nodeNames;
}
/**
* @return how long to wait after adding the exclusions for the nodes to be removed from the voting configuration.
*/
@ -122,14 +218,20 @@ public class AddVotingConfigExclusionsRequest extends MasterNodeRequest<AddVotin
public void writeTo(StreamOutput out) throws IOException {
super.writeTo(out);
out.writeStringArray(nodeDescriptions);
if (out.getVersion().onOrAfter(Version.V_7_8_0)) {
out.writeStringArray(nodeIds);
out.writeStringArray(nodeNames);
}
out.writeTimeValue(timeout);
}
@Override
public String toString() {
return "AddVotingConfigExclusionsRequest{" +
"nodeDescriptions=" + Arrays.asList(nodeDescriptions) +
", timeout=" + timeout +
"nodeDescriptions=" + Arrays.asList(nodeDescriptions) + ", " +
"nodeIds=" + Arrays.asList(nodeIds) + ", " +
"nodeNames=" + Arrays.asList(nodeNames) + ", " +
"timeout=" + timeout +
'}';
}
}

View File

@ -229,6 +229,7 @@ public class CoordinationMetadata implements Writeable, ToXContentFragment {
}
public static class VotingConfigExclusion implements Writeable, ToXContentFragment {
public static final String MISSING_VALUE_MARKER = "_absent_";
private final String nodeId;
private final String nodeName;

View File

@ -898,6 +898,7 @@ public class Coordinator extends AbstractLifecycleComponent implements Discovery
// Package-private for testing
ClusterState improveConfiguration(ClusterState clusterState) {
assert Thread.holdsLock(mutex) : "Coordinator mutex not held";
assert validVotingConfigExclusionState(clusterState) : clusterState;
// exclude any nodes whose ID is in the voting config exclusions list ...
final Stream<String> excludedNodeIds = clusterState.getVotingConfigExclusions().stream().map(VotingConfigExclusion::getNodeId);
@ -928,6 +929,31 @@ public class Coordinator extends AbstractLifecycleComponent implements Discovery
return clusterState;
}
/*
* Valid Voting Configuration Exclusion state criteria:
* 1. Every voting config exclusion with an ID of _absent_ should not match any nodes currently in the cluster by name
* 2. Every voting config exclusion with a name of _absent_ should not match any nodes currently in the cluster by ID
*/
static boolean validVotingConfigExclusionState(ClusterState clusterState) {
Set<VotingConfigExclusion> votingConfigExclusions = clusterState.getVotingConfigExclusions();
Set<String> nodeNamesWithAbsentId = votingConfigExclusions.stream()
.filter(e -> e.getNodeId().equals(VotingConfigExclusion.MISSING_VALUE_MARKER))
.map(VotingConfigExclusion::getNodeName)
.collect(Collectors.toSet());
Set<String> nodeIdsWithAbsentName = votingConfigExclusions.stream()
.filter(e -> e.getNodeName().equals(VotingConfigExclusion.MISSING_VALUE_MARKER))
.map(VotingConfigExclusion::getNodeId)
.collect(Collectors.toSet());
for (DiscoveryNode node : clusterState.getNodes()) {
if (node.isMasterNode() &&
(nodeIdsWithAbsentName.contains(node.getId()) || nodeNamesWithAbsentId.contains(node.getName()))) {
return false;
}
}
return true;
}
private AtomicBoolean reconfigurationTaskScheduled = new AtomicBoolean();
private void scheduleReconfigurationIfNeeded() {

View File

@ -39,8 +39,12 @@ import org.elasticsearch.persistent.PersistentTasksCustomMetadata;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.function.BiConsumer;
import java.util.stream.Collectors;
import static org.elasticsearch.gateway.GatewayService.STATE_NOT_RECOVERED_BLOCK;
@ -129,6 +133,7 @@ public class JoinTaskExecutor implements ClusterStateTaskExecutor<JoinTaskExecut
// we only enforce major version transitions on a fully formed clusters
final boolean enforceMajorVersion = currentState.getBlocks().hasGlobalBlock(STATE_NOT_RECOVERED_BLOCK) == false;
// processing any joins
Map<String, String> joiniedNodeNameIds = new HashMap<>();
for (final Task joinTask : joiningNodes) {
if (joinTask.isBecomeMasterTask() || joinTask.isFinishElectionTask()) {
// noop
@ -148,6 +153,9 @@ public class JoinTaskExecutor implements ClusterStateTaskExecutor<JoinTaskExecut
nodesChanged = true;
minClusterNodeVersion = Version.min(minClusterNodeVersion, node.getVersion());
maxClusterNodeVersion = Version.max(maxClusterNodeVersion, node.getVersion());
if (node.isMasterNode()) {
joiniedNodeNameIds.put(node.getName(), node.getId());
}
} catch (IllegalArgumentException | IllegalStateException e) {
results.failure(joinTask, e);
continue;
@ -155,11 +163,37 @@ public class JoinTaskExecutor implements ClusterStateTaskExecutor<JoinTaskExecut
}
results.success(joinTask);
}
if (nodesChanged) {
rerouteService.reroute("post-join reroute", Priority.HIGH, ActionListener.wrap(
r -> logger.trace("post-join reroute completed"),
e -> logger.debug("post-join reroute failed", e)));
if (joiniedNodeNameIds.isEmpty() == false) {
Set<CoordinationMetadata.VotingConfigExclusion> currentVotingConfigExclusions = currentState.getVotingConfigExclusions();
Set<CoordinationMetadata.VotingConfigExclusion> newVotingConfigExclusions = currentVotingConfigExclusions.stream()
.map(e -> {
// Update nodeId in VotingConfigExclusion when a new node with excluded node name joins
if (CoordinationMetadata.VotingConfigExclusion.MISSING_VALUE_MARKER.equals(e.getNodeId()) &&
joiniedNodeNameIds.containsKey(e.getNodeName())) {
return new CoordinationMetadata.VotingConfigExclusion(joiniedNodeNameIds.get(e.getNodeName()), e.getNodeName());
} else {
return e;
}
}).collect(Collectors.toSet());
// if VotingConfigExclusions did get updated
if (newVotingConfigExclusions.equals(currentVotingConfigExclusions) == false) {
CoordinationMetadata.Builder coordMetadataBuilder = CoordinationMetadata.builder(currentState.coordinationMetadata())
.clearVotingConfigExclusions();
newVotingConfigExclusions.forEach(coordMetadataBuilder::addVotingConfigExclusion);
Metadata newMetadata = Metadata.builder(currentState.metadata())
.coordinationMetadata(coordMetadataBuilder.build()).build();
return results.build(allocationService.adaptAutoExpandReplicas(newState.nodes(nodesBuilder)
.metadata(newMetadata).build()));
}
}
return results.build(allocationService.adaptAutoExpandReplicas(newState.nodes(nodesBuilder).build()));
} else {
// we must return a new cluster state instance to force publishing. This is important

View File

@ -19,6 +19,8 @@
package org.elasticsearch.rest.action.admin.cluster;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
import org.elasticsearch.action.admin.cluster.configuration.AddVotingConfigExclusionsAction;
import org.elasticsearch.action.admin.cluster.configuration.AddVotingConfigExclusionsRequest;
import org.elasticsearch.client.node.NodeClient;
@ -29,14 +31,19 @@ import org.elasticsearch.rest.RestRequest;
import org.elasticsearch.rest.action.RestToXContentListener;
import java.io.IOException;
import java.util.Arrays;
import java.util.List;
import static java.util.Collections.singletonList;
import static org.elasticsearch.rest.RestRequest.Method.POST;
public class RestAddVotingConfigExclusionAction extends BaseRestHandler {
private static final TimeValue DEFAULT_TIMEOUT = TimeValue.timeValueSeconds(30L);
private static final Logger logger = LogManager.getLogger(RestAddVotingConfigExclusionAction.class);
private static final String DEPRECATION_MESSAGE = "POST /_cluster/voting_config_exclusions/{node_name} " +
"will be removed in a future version. " +
"Please use POST /_cluster/voting_config_exclusions?node_ids=... " +
"or POST /_cluster/voting_config_exclusions?node_names=... instead.";
@Override
public String getName() {
@ -45,7 +52,9 @@ public class RestAddVotingConfigExclusionAction extends BaseRestHandler {
@Override
public List<Route> routes() {
return singletonList(new Route(POST, "/_cluster/voting_config_exclusions/{node_name}"));
return Arrays.asList(
new DeprecatedRoute(POST, "/_cluster/voting_config_exclusions/{node_name}", DEPRECATION_MESSAGE),
new Route(POST, "/_cluster/voting_config_exclusions"));
}
@Override
@ -59,10 +68,29 @@ public class RestAddVotingConfigExclusionAction extends BaseRestHandler {
}
AddVotingConfigExclusionsRequest resolveVotingConfigExclusionsRequest(final RestRequest request) {
String nodeName = request.param("node_name");
String deprecatedNodeDescription = null;
String nodeIds = null;
String nodeNames = null;
if (request.hasParam("node_name")) {
deprecatedNodeDescription = request.param("node_name");
}
if (request.hasParam("node_ids")){
nodeIds = request.param("node_ids");
}
if (request.hasParam("node_names")){
nodeNames = request.param("node_names");
}
return new AddVotingConfigExclusionsRequest(
Strings.splitStringByCommaToArray(nodeName),
Strings.splitStringByCommaToArray(deprecatedNodeDescription),
Strings.splitStringByCommaToArray(nodeIds),
Strings.splitStringByCommaToArray(nodeNames),
TimeValue.parseTimeValue(request.param("timeout"), DEFAULT_TIMEOUT, getClass().getSimpleName() + ".timeout")
);
}
}

View File

@ -27,31 +27,57 @@ import org.elasticsearch.cluster.metadata.Metadata;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes.Builder;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.unit.TimeValue;
import org.elasticsearch.test.ESTestCase;
import java.io.IOException;
import java.util.Collections;
import static java.util.Collections.emptyMap;
import static java.util.Collections.emptySet;
import static java.util.Collections.singleton;
import static org.hamcrest.Matchers.contains;
import static org.hamcrest.Matchers.containsInAnyOrder;
import static org.hamcrest.Matchers.equalTo;
public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
private static final String NODE_IDENTIFIERS_INCORRECTLY_SET_MSG = "Please set node identifiers correctly. " +
"One and only one of [node_name], [node_names] and [node_ids] has to be set";
public void testSerialization() throws IOException {
int descriptionCount = between(0, 5);
int descriptionCount = between(1, 5);
String[] descriptions = new String[descriptionCount];
for (int i = 0; i < descriptionCount; i++) {
descriptions[i] = randomAlphaOfLength(10);
}
TimeValue timeout = TimeValue.timeValueMillis(between(0, 30000));
final AddVotingConfigExclusionsRequest originalRequest = new AddVotingConfigExclusionsRequest(descriptions, timeout);
final AddVotingConfigExclusionsRequest originalRequest = new AddVotingConfigExclusionsRequest(descriptions, Strings.EMPTY_ARRAY,
Strings.EMPTY_ARRAY, timeout);
final AddVotingConfigExclusionsRequest deserialized = copyWriteable(originalRequest, writableRegistry(),
AddVotingConfigExclusionsRequest::new);
assertThat(deserialized.getNodeDescriptions(), equalTo(originalRequest.getNodeDescriptions()));
assertThat(deserialized.getTimeout(), equalTo(originalRequest.getTimeout()));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testSerializationForNodeIdOrNodeName() throws IOException {
AddVotingConfigExclusionsRequest originalRequest = new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY,
new String[]{"nodeId1", "nodeId2"}, Strings.EMPTY_ARRAY, TimeValue.ZERO);
AddVotingConfigExclusionsRequest deserialized = copyWriteable(originalRequest, writableRegistry(),
AddVotingConfigExclusionsRequest::new);
assertThat(deserialized.getNodeDescriptions(), equalTo(originalRequest.getNodeDescriptions()));
assertThat(deserialized.getNodeIds(), equalTo(originalRequest.getNodeIds()));
assertThat(deserialized.getNodeNames(), equalTo(originalRequest.getNodeNames()));
assertThat(deserialized.getTimeout(), equalTo(originalRequest.getTimeout()));
originalRequest = new AddVotingConfigExclusionsRequest("nodeName1", "nodeName2");
deserialized = copyWriteable(originalRequest, writableRegistry(), AddVotingConfigExclusionsRequest::new);
assertThat(deserialized.getNodeDescriptions(), equalTo(originalRequest.getNodeDescriptions()));
assertThat(deserialized.getNodeIds(), equalTo(originalRequest.getNodeIds()));
assertThat(deserialized.getNodeNames(), equalTo(originalRequest.getNodeNames()));
assertThat(deserialized.getTimeout(), equalTo(originalRequest.getTimeout()));
}
public void testResolve() {
@ -60,7 +86,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"local",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion localNodeExclusion = new VotingConfigExclusion(localNode);
final DiscoveryNode otherNode1 = new DiscoveryNode(
@ -68,7 +94,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"other1",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode1Exclusion = new VotingConfigExclusion(otherNode1);
final DiscoveryNode otherNode2 = new DiscoveryNode(
@ -76,7 +102,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"other2",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode2Exclusion = new VotingConfigExclusion(otherNode2);
final DiscoveryNode otherDataNode
@ -85,18 +111,163 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
final ClusterState clusterState = ClusterState.builder(new ClusterName("cluster")).nodes(new Builder()
.add(localNode).add(otherNode1).add(otherNode2).add(otherDataNode).localNodeId(localNode.getId())).build();
assertThat(makeRequest().resolveVotingConfigExclusions(clusterState),
assertThat(makeRequestWithNodeDescriptions("_all").resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(localNodeExclusion, otherNode1Exclusion, otherNode2Exclusion));
assertThat(makeRequest("_all").resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(localNodeExclusion, otherNode1Exclusion, otherNode2Exclusion));
assertThat(makeRequest("_local").resolveVotingConfigExclusions(clusterState),
assertThat(makeRequestWithNodeDescriptions("_local").resolveVotingConfigExclusions(clusterState),
contains(localNodeExclusion));
assertThat(makeRequest("other*").resolveVotingConfigExclusions(clusterState),
assertThat(makeRequestWithNodeDescriptions("other*").resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(otherNode1Exclusion, otherNode2Exclusion));
assertThat(expectThrows(IllegalArgumentException.class,
() -> makeRequest("not-a-node").resolveVotingConfigExclusions(clusterState)).getMessage(),
equalTo("add voting config exclusions request for [not-a-node] matched no master-eligible nodes"));
() -> makeRequestWithNodeDescriptions("not-a-node").resolveVotingConfigExclusions(clusterState)).getMessage(),
equalTo("add voting config exclusions request for [not-a-node] matched no master-eligible nodes"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testResolveAllNodeIdentifiersNullOrEmpty() {
assertThat(expectThrows(IllegalArgumentException.class,
() -> new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, Strings.EMPTY_ARRAY,
Strings.EMPTY_ARRAY, TimeValue.ZERO)).getMessage(),
equalTo(NODE_IDENTIFIERS_INCORRECTLY_SET_MSG));
}
public void testResolveMoreThanOneNodeIdentifiersSet() {
assertThat(expectThrows(IllegalArgumentException.class,
() -> new AddVotingConfigExclusionsRequest(new String[]{"local"}, new String[]{"nodeId"},
Strings.EMPTY_ARRAY, TimeValue.ZERO)).getMessage(),
equalTo(NODE_IDENTIFIERS_INCORRECTLY_SET_MSG));
assertThat(expectThrows(IllegalArgumentException.class,
() -> new AddVotingConfigExclusionsRequest(new String[]{"local"}, Strings.EMPTY_ARRAY,
new String[]{"nodeName"}, TimeValue.ZERO)).getMessage(),
equalTo(NODE_IDENTIFIERS_INCORRECTLY_SET_MSG));
assertThat(expectThrows(IllegalArgumentException.class,
() -> new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"nodeId"},
new String[]{"nodeName"}, TimeValue.ZERO)).getMessage(),
equalTo(NODE_IDENTIFIERS_INCORRECTLY_SET_MSG));
assertThat(expectThrows(IllegalArgumentException.class,
() -> new AddVotingConfigExclusionsRequest(new String[]{"local"}, new String[]{"nodeId"},
new String[]{"nodeName"}, TimeValue.ZERO)).getMessage(),
equalTo(NODE_IDENTIFIERS_INCORRECTLY_SET_MSG));
}
public void testResolveByNodeIds() {
final DiscoveryNode node1 = new DiscoveryNode(
"nodeName1",
"nodeId1",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion node1Exclusion = new VotingConfigExclusion(node1);
final DiscoveryNode node2 = new DiscoveryNode(
"nodeName2",
"nodeId2",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion node2Exclusion = new VotingConfigExclusion(node2);
final DiscoveryNode node3 = new DiscoveryNode(
"nodeName3",
"nodeId3",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion unresolvableVotingConfigExclusion = new VotingConfigExclusion("unresolvableNodeId",
VotingConfigExclusion.MISSING_VALUE_MARKER);
final ClusterState clusterState = ClusterState.builder(new ClusterName("cluster"))
.nodes(new Builder().add(node1).add(node2).add(node3).localNodeId(node1.getId())).build();
assertThat(new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"nodeId1", "nodeId2"},
Strings.EMPTY_ARRAY, TimeValue.ZERO).resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(node1Exclusion, node2Exclusion));
assertThat(new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"nodeId1", "unresolvableNodeId"},
Strings.EMPTY_ARRAY, TimeValue.ZERO).resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(node1Exclusion, unresolvableVotingConfigExclusion));
}
public void testResolveByNodeNames() {
final DiscoveryNode node1 = new DiscoveryNode("nodeName1",
"nodeId1",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion node1Exclusion = new VotingConfigExclusion(node1);
final DiscoveryNode node2 = new DiscoveryNode("nodeName2",
"nodeId2",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion node2Exclusion = new VotingConfigExclusion(node2);
final DiscoveryNode node3 = new DiscoveryNode("nodeName3",
"nodeId3",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion unresolvableVotingConfigExclusion = new VotingConfigExclusion(
VotingConfigExclusion.MISSING_VALUE_MARKER, "unresolvableNodeName");
final ClusterState clusterState = ClusterState.builder(new ClusterName("cluster"))
.nodes(new Builder().add(node1).add(node2).add(node3).localNodeId(node1.getId())).build();
assertThat(new AddVotingConfigExclusionsRequest("nodeName1", "nodeName2").resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(node1Exclusion, node2Exclusion));
assertThat(new AddVotingConfigExclusionsRequest("nodeName1", "unresolvableNodeName").resolveVotingConfigExclusions(clusterState),
containsInAnyOrder(node1Exclusion, unresolvableVotingConfigExclusion));
}
public void testResolveRemoveExistingVotingConfigExclusions() {
final DiscoveryNode node1 = new DiscoveryNode("nodeName1",
"nodeId1",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final DiscoveryNode node2 = new DiscoveryNode("nodeName2",
"nodeId2",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion node2Exclusion = new VotingConfigExclusion(node2);
final DiscoveryNode node3 = new DiscoveryNode("nodeName3",
"nodeId3",
buildNewFakeTransportAddress(),
emptyMap(),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion existingVotingConfigExclusion = new VotingConfigExclusion(node1);
Metadata metadata = Metadata.builder()
.coordinationMetadata(CoordinationMetadata.builder()
.addVotingConfigExclusion(existingVotingConfigExclusion).build())
.build();
final ClusterState clusterState = ClusterState.builder(new ClusterName("cluster")).metadata(metadata)
.nodes(new Builder().add(node1).add(node2).add(node3).localNodeId(node1.getId())).build();
assertThat(new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"nodeId1", "nodeId2"},
Strings.EMPTY_ARRAY, TimeValue.ZERO).resolveVotingConfigExclusions(clusterState),
contains(node2Exclusion));
}
public void testResolveAndCheckMaximum() {
@ -105,7 +276,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"local",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion localNodeExclusion = new VotingConfigExclusion(localNode);
final DiscoveryNode otherNode1 = new DiscoveryNode(
@ -113,7 +284,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"other1",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode1Exclusion = new VotingConfigExclusion(otherNode1);
final DiscoveryNode otherNode2 = new DiscoveryNode(
@ -121,7 +292,7 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
"other2",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode2Exclusion = new VotingConfigExclusion(otherNode2);
@ -131,22 +302,18 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
.coordinationMetadata(CoordinationMetadata.builder().addVotingConfigExclusion(otherNode1Exclusion).build()));
final ClusterState clusterState = builder.build();
assertThat(makeRequest().resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 3, "setting.name"),
containsInAnyOrder(localNodeExclusion, otherNode2Exclusion));
assertThat(makeRequest("_local").resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 2, "setting.name"),
assertThat(makeRequestWithNodeDescriptions("_local").resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 2, "setting.name"),
contains(localNodeExclusion));
assertThat(expectThrows(IllegalArgumentException.class,
() -> makeRequest().resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 2, "setting.name")).getMessage(),
equalTo("add voting config exclusions request for [] would add [2] exclusions to the existing [1] which would exceed " +
"the maximum of [2] set by [setting.name]"));
assertThat(expectThrows(IllegalArgumentException.class,
() -> makeRequest("_local").resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 1, "setting.name")).getMessage(),
() -> makeRequestWithNodeDescriptions("_local").resolveVotingConfigExclusionsAndCheckMaximum(clusterState, 1, "setting.name"))
.getMessage(),
equalTo("add voting config exclusions request for [_local] would add [1] exclusions to the existing [1] which would " +
"exceed the maximum of [1] set by [setting.name]"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
private static AddVotingConfigExclusionsRequest makeRequest(String... descriptions) {
return new AddVotingConfigExclusionsRequest(descriptions);
private static AddVotingConfigExclusionsRequest makeRequestWithNodeDescriptions(String... nodeDescriptions) {
return new AddVotingConfigExclusionsRequest(nodeDescriptions, Strings.EMPTY_ARRAY,
Strings.EMPTY_ARRAY, TimeValue.timeValueSeconds(30));
}
}

View File

@ -36,6 +36,7 @@ import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes.Builder;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.settings.ClusterSettings;
import org.elasticsearch.common.settings.Settings;
@ -53,7 +54,6 @@ import org.junit.Before;
import org.junit.BeforeClass;
import java.io.IOException;
import java.util.Collections;
import java.util.HashSet;
import java.util.Set;
import java.util.concurrent.CountDownLatch;
@ -62,6 +62,7 @@ import java.util.function.Consumer;
import static java.util.Collections.emptyMap;
import static java.util.Collections.emptySet;
import static java.util.Collections.singleton;
import static org.elasticsearch.action.admin.cluster.configuration.TransportAddVotingConfigExclusionsAction.MAXIMUM_VOTING_CONFIG_EXCLUSIONS_SETTING;
import static org.elasticsearch.cluster.ClusterState.builder;
import static org.elasticsearch.test.ClusterServiceUtils.createClusterService;
@ -104,7 +105,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
name,
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
}
@ -152,8 +153,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other1"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, new AddVotingConfigExclusionsRequest("other1"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -169,7 +169,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other1", "other2"}),
new AddVotingConfigExclusionsRequest("other1", "other2"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -185,8 +185,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other*"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, makeRequestWithNodeDescriptions("other*"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -196,14 +195,14 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
containsInAnyOrder(otherNode1Exclusion, otherNode2Exclusion));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testWithdrawsVotesFromAllMasterEligibleNodes() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"_all"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, makeRequestWithNodeDescriptions("_all"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -213,14 +212,14 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
containsInAnyOrder(localNodeExclusion, otherNode1Exclusion, otherNode2Exclusion));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testWithdrawsVoteFromLocalNode() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"_local"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, makeRequestWithNodeDescriptions("_local"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -230,6 +229,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
contains(localNodeExclusion));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testReturnsImmediatelyIfVoteAlreadyWithdrawn() throws InterruptedException {
@ -244,8 +244,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final CountDownLatch countDownLatch = new CountDownLatch(1);
// no observer to reconfigure
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other1"}, TimeValue.ZERO),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, new AddVotingConfigExclusionsRequest("other1"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -257,12 +256,11 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
contains(otherNode1Exclusion));
}
public void testReturnsErrorIfNoMatchingNodes() throws InterruptedException {
public void testReturnsErrorIfNoMatchingNodeDescriptions() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
final SetOnce<TransportException> exceptionHolder = new SetOnce<>();
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"not-a-node"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, makeRequestWithNodeDescriptions("not-a-node"),
expectError(e -> {
exceptionHolder.set(e);
countDownLatch.countDown();
@ -274,6 +272,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertThat(rootCause, instanceOf(IllegalArgumentException.class));
assertThat(rootCause.getMessage(),
equalTo("add voting config exclusions request for [not-a-node] matched no master-eligible nodes"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testOnlyMatchesMasterEligibleNodes() throws InterruptedException {
@ -281,7 +280,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final SetOnce<TransportException> exceptionHolder = new SetOnce<>();
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"_all", "master:false"}),
makeRequestWithNodeDescriptions("_all", "master:false"),
expectError(e -> {
exceptionHolder.set(e);
countDownLatch.countDown();
@ -293,6 +292,74 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertThat(rootCause, instanceOf(IllegalArgumentException.class));
assertThat(rootCause.getMessage(),
equalTo("add voting config exclusions request for [_all, master:false] matched no master-eligible nodes"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testExcludeAbsentNodesByNodeIds() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"absent_id"},
Strings.EMPTY_ARRAY, TimeValue.timeValueSeconds(30)),
expectSuccess(e -> {
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertEquals(singleton(new VotingConfigExclusion("absent_id", VotingConfigExclusion.MISSING_VALUE_MARKER)),
clusterService.getClusterApplierService().state().getVotingConfigExclusions());
}
public void testExcludeExistingNodesByNodeIds() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"other1", "other2"},
Strings.EMPTY_ARRAY, TimeValue.timeValueSeconds(30)),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
containsInAnyOrder(otherNode1Exclusion, otherNode2Exclusion));
}
public void testExcludeAbsentNodesByNodeNames() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, new AddVotingConfigExclusionsRequest("absent_node"),
expectSuccess(e -> {
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertEquals(singleton(new VotingConfigExclusion(VotingConfigExclusion.MISSING_VALUE_MARKER, "absent_node")),
clusterService.getClusterApplierService().state().getVotingConfigExclusions());
}
public void testExcludeExistingNodesByNodeNames() throws InterruptedException {
final CountDownLatch countDownLatch = new CountDownLatch(1);
clusterStateObserver.waitForNextChange(new AdjustConfigurationForExclusions());
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest("other1", "other2"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
containsInAnyOrder(otherNode1Exclusion, otherNode2Exclusion));
}
public void testSucceedsEvenIfAllExclusionsAlreadyAdded() throws InterruptedException {
@ -307,8 +374,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final CountDownLatch countDownLatch = new CountDownLatch(1);
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other1"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, new AddVotingConfigExclusionsRequest("other1"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
@ -320,6 +386,56 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
contains(otherNode1Exclusion));
}
public void testExcludeByNodeIdSucceedsEvenIfAllExclusionsAlreadyAdded() throws InterruptedException {
final ClusterState state = clusterService.state();
final ClusterState.Builder builder = builder(state);
builder.metadata(Metadata.builder(state.metadata()).
coordinationMetadata(
CoordinationMetadata.builder(state.coordinationMetadata())
.addVotingConfigExclusion(otherNode1Exclusion).
build()));
setState(clusterService, builder);
final CountDownLatch countDownLatch = new CountDownLatch(1);
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(Strings.EMPTY_ARRAY, new String[]{"other1"},
Strings.EMPTY_ARRAY, TimeValue.timeValueSeconds(30)),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
contains(otherNode1Exclusion));
}
public void testExcludeByNodeNameSucceedsEvenIfAllExclusionsAlreadyAdded() throws InterruptedException {
final ClusterState state = clusterService.state();
final ClusterState.Builder builder = builder(state);
builder.metadata(Metadata.builder(state.metadata()).
coordinationMetadata(
CoordinationMetadata.builder(state.coordinationMetadata())
.addVotingConfigExclusion(otherNode1Exclusion).
build()));
setState(clusterService, builder);
final CountDownLatch countDownLatch = new CountDownLatch(1);
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, new AddVotingConfigExclusionsRequest("other1"),
expectSuccess(r -> {
assertNotNull(r);
countDownLatch.countDown();
})
);
assertTrue(countDownLatch.await(30, TimeUnit.SECONDS));
assertThat(clusterService.getClusterApplierService().state().getVotingConfigExclusions(),
contains(otherNode1Exclusion));
}
public void testReturnsErrorIfMaximumExclusionCountExceeded() throws InterruptedException {
final Metadata.Builder metadataBuilder = Metadata.builder(clusterService.state().metadata());
CoordinationMetadata.Builder coordinationMetadataBuilder =
@ -358,8 +474,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final CountDownLatch countDownLatch = new CountDownLatch(1);
final SetOnce<TransportException> exceptionHolder = new SetOnce<>();
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other*"}),
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME, makeRequestWithNodeDescriptions("other*"),
expectError(e -> {
exceptionHolder.set(e);
countDownLatch.countDown();
@ -372,6 +487,7 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
assertThat(rootCause.getMessage(), equalTo("add voting config exclusions request for [other*] would add [" + newCount +
"] exclusions to the existing [" + existingCount +
"] which would exceed the maximum of [" + actualMaximum + "] set by [cluster.max_voting_config_exclusions]"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
public void testTimesOut() throws InterruptedException {
@ -379,7 +495,8 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final SetOnce<TransportException> exceptionHolder = new SetOnce<>();
transportService.sendRequest(localNode, AddVotingConfigExclusionsAction.NAME,
new AddVotingConfigExclusionsRequest(new String[]{"other1"}, TimeValue.timeValueMillis(100)),
new AddVotingConfigExclusionsRequest(new String[]{"other1"}, Strings.EMPTY_ARRAY, Strings.EMPTY_ARRAY,
TimeValue.timeValueMillis(100)),
expectError(e -> {
exceptionHolder.set(e);
countDownLatch.countDown();
@ -390,6 +507,8 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
final Throwable rootCause = exceptionHolder.get().getRootCause();
assertThat(rootCause,instanceOf(ElasticsearchTimeoutException.class));
assertThat(rootCause.getMessage(), startsWith("timed out waiting for voting config exclusions [{other1}"));
assertWarnings(AddVotingConfigExclusionsRequest.DEPRECATION_MESSAGE);
}
private TransportResponseHandler<AddVotingConfigExclusionsResponse> expectSuccess(
@ -467,4 +586,10 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
throw new AssertionError("unexpected timeout");
}
}
private AddVotingConfigExclusionsRequest makeRequestWithNodeDescriptions(String... nodeDescriptions) {
return new AddVotingConfigExclusionsRequest(nodeDescriptions, Strings.EMPTY_ARRAY,
Strings.EMPTY_ARRAY, TimeValue.timeValueSeconds(30));
}
}

View File

@ -123,7 +123,7 @@ public class MinimumMasterNodesIT extends ESIntegTestCase {
String masterNode = internalCluster().getMasterName();
String otherNode = node1Name.equals(masterNode) ? node2Name : node1Name;
logger.info("--> add voting config exclusion for non-master node, to be sure it's not elected");
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(new String[]{otherNode})).get();
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(otherNode)).get();
logger.info("--> stop master node, no master block should appear");
Settings masterDataPathSettings = internalCluster().dataPathSettings(masterNode);
internalCluster().stopRandomNode(InternalTestCluster.nameFilter(masterNode));
@ -170,7 +170,7 @@ public class MinimumMasterNodesIT extends ESIntegTestCase {
masterNode = internalCluster().getMasterName();
otherNode = node1Name.equals(masterNode) ? node2Name : node1Name;
logger.info("--> add voting config exclusion for master node, to be sure it's not elected");
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(new String[]{masterNode})).get();
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(masterNode)).get();
logger.info("--> stop non-master node, no master block should appear");
Settings otherNodeDataPathSettings = internalCluster().dataPathSettings(otherNode);
internalCluster().stopRandomNode(InternalTestCluster.nameFilter(otherNode));

View File

@ -115,8 +115,7 @@ public class SpecificMasterNodesIT extends ESIntegTestCase {
.execute().actionGet().getState().nodes().getMasterNode().getName(), equalTo(masterNodeName));
logger.info("--> closing master node (1)");
client().execute(AddVotingConfigExclusionsAction.INSTANCE,
new AddVotingConfigExclusionsRequest(new String[]{masterNodeName})).get();
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(masterNodeName)).get();
// removing the master from the voting configuration immediately triggers the master to step down
assertBusy(() -> {
assertThat(internalCluster().nonMasterClient().admin().cluster().prepareState()

View File

@ -32,6 +32,8 @@ import org.elasticsearch.cluster.coordination.CoordinationMetadata.VotingConfigu
import org.elasticsearch.cluster.coordination.Coordinator.Mode;
import org.elasticsearch.cluster.metadata.Metadata;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.logging.Loggers;
import org.elasticsearch.common.regex.Regex;
@ -56,6 +58,8 @@ import java.util.concurrent.atomic.AtomicBoolean;
import java.util.function.Function;
import java.util.stream.Collectors;
import static java.util.Collections.emptyMap;
import static java.util.Collections.singleton;
import static org.elasticsearch.cluster.coordination.AbstractCoordinatorTestCase.Cluster.DEFAULT_DELAY_VARIABILITY;
import static org.elasticsearch.cluster.coordination.AbstractCoordinatorTestCase.Cluster.EXTREME_DELAY_VARIABILITY;
import static org.elasticsearch.cluster.coordination.Coordinator.Mode.CANDIDATE;
@ -1434,4 +1438,44 @@ public class CoordinatorTests extends AbstractCoordinatorTestCase {
}
}
public void testImproveConfigurationPerformsVotingConfigExclusionStateCheck() {
try (Cluster cluster = new Cluster(1)) {
cluster.runRandomly();
cluster.stabilise();
final Coordinator coordinator = cluster.getAnyLeader().coordinator;
final ClusterState currentState = coordinator.getLastAcceptedState();
Set<CoordinationMetadata.VotingConfigExclusion> newVotingConfigExclusion1 =
singleton(new CoordinationMetadata.VotingConfigExclusion("resolvableNodeId",
CoordinationMetadata.VotingConfigExclusion.MISSING_VALUE_MARKER));
ClusterState newState1 = buildNewClusterStateWithVotingConfigExclusion(currentState, newVotingConfigExclusion1);
assertFalse(Coordinator.validVotingConfigExclusionState(newState1));
Set<CoordinationMetadata.VotingConfigExclusion> newVotingConfigExclusion2 =
singleton(new CoordinationMetadata.VotingConfigExclusion(
CoordinationMetadata.VotingConfigExclusion.MISSING_VALUE_MARKER, "resolvableNodeName"));
ClusterState newState2 = buildNewClusterStateWithVotingConfigExclusion(currentState, newVotingConfigExclusion2);
assertFalse(Coordinator.validVotingConfigExclusionState(newState2));
}
}
private ClusterState buildNewClusterStateWithVotingConfigExclusion(ClusterState currentState,
Set<CoordinationMetadata.VotingConfigExclusion> newVotingConfigExclusion) {
DiscoveryNodes newNodes = DiscoveryNodes.builder(currentState.nodes())
.add(new DiscoveryNode("resolvableNodeName", "resolvableNodeId", buildNewFakeTransportAddress(),
emptyMap(), singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT))
.build();
CoordinationMetadata.Builder coordMetadataBuilder = CoordinationMetadata.builder(currentState.coordinationMetadata());
newVotingConfigExclusion.forEach(coordMetadataBuilder::addVotingConfigExclusion);
Metadata newMetadata = Metadata.builder(currentState.metadata()).coordinationMetadata(coordMetadataBuilder.build()).build();
return ClusterState.builder(currentState).nodes(newNodes).metadata(newMetadata).build();
}
}

View File

@ -71,6 +71,7 @@ import java.util.stream.Stream;
import static java.util.Collections.emptyList;
import static java.util.Collections.emptyMap;
import static java.util.Collections.singleton;
import static java.util.Collections.singletonList;
import static org.elasticsearch.transport.TransportService.HANDSHAKE_ACTION_NAME;
import static org.hamcrest.Matchers.containsString;
@ -193,7 +194,7 @@ public class NodeJoinTests extends ESTestCase {
protected DiscoveryNode newNode(int i, boolean master) {
final Set<DiscoveryNodeRole> roles;
if (master) {
roles = Collections.singleton(DiscoveryNodeRole.MASTER_ROLE);
roles = singleton(DiscoveryNodeRole.MASTER_ROLE);
} else {
roles = Collections.emptySet();
}
@ -393,6 +394,45 @@ public class NodeJoinTests extends ESTestCase {
assertTrue(isLocalNodeElectedMaster());
}
public void testJoinUpdateVotingConfigExclusion() throws Exception {
DiscoveryNode initialNode = newNode(0, true);
long initialTerm = randomLongBetween(1, 10);
long initialVersion = randomLongBetween(1, 10);
CoordinationMetadata.VotingConfigExclusion votingConfigExclusion = new CoordinationMetadata.VotingConfigExclusion(
CoordinationMetadata.VotingConfigExclusion.MISSING_VALUE_MARKER, "knownNodeName");
setupFakeMasterServiceAndCoordinator(initialTerm, buildStateWithVotingConfigExclusion(initialNode, initialTerm,
initialVersion, votingConfigExclusion));
DiscoveryNode knownJoiningNode = new DiscoveryNode("knownNodeName", "newNodeId", buildNewFakeTransportAddress(),
emptyMap(), singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
long newTerm = initialTerm + randomLongBetween(1, 10);
long newerTerm = newTerm + randomLongBetween(1, 10);
joinNodeAndRun(new JoinRequest(knownJoiningNode, initialTerm,
Optional.of(new Join(knownJoiningNode, initialNode, newerTerm, initialTerm, initialVersion))));
assertTrue(MasterServiceTests.discoveryState(masterService).getVotingConfigExclusions().stream().anyMatch(exclusion -> {
return "knownNodeName".equals(exclusion.getNodeName()) && "newNodeId".equals(exclusion.getNodeId());
}));
}
private ClusterState buildStateWithVotingConfigExclusion(DiscoveryNode initialNode,
long initialTerm,
long initialVersion,
CoordinationMetadata.VotingConfigExclusion votingConfigExclusion) {
ClusterState initialState = initialState(initialNode, initialTerm, initialVersion,
new VotingConfiguration(singleton(initialNode.getId())));
Metadata newMetadata = Metadata.builder(initialState.metadata())
.coordinationMetadata(CoordinationMetadata.builder(initialState.coordinationMetadata())
.addVotingConfigExclusion(votingConfigExclusion)
.build())
.build();
return ClusterState.builder(initialState).metadata(newMetadata).build();
}
private void handleStartJoinFrom(DiscoveryNode node, long term) throws Exception {
final RequestHandlerRegistry<StartJoinRequest> startJoinHandler = (RequestHandlerRegistry<StartJoinRequest>)
transport.getRequestHandler(JoinHelper.START_JOIN_ACTION_NAME);

View File

@ -54,8 +54,7 @@ public class VotingConfigurationIT extends ESIntegTestCase {
final String originalMaster = internalCluster().getMasterName();
logger.info("--> excluding master node {}", originalMaster);
client().execute(AddVotingConfigExclusionsAction.INSTANCE,
new AddVotingConfigExclusionsRequest(new String[]{originalMaster})).get();
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(originalMaster)).get();
client().admin().cluster().prepareHealth().setWaitForEvents(Priority.LANGUID).get();
assertNotEquals(originalMaster, internalCluster().getMasterName());
}

View File

@ -310,7 +310,7 @@ public class RecoveryFromGatewayIT extends ESIntegTestCase {
Map<String, long[]> primaryTerms = assertAndCapturePrimaryTerms(null);
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(new String[]{firstNode})).get();
client().execute(AddVotingConfigExclusionsAction.INSTANCE, new AddVotingConfigExclusionsRequest(firstNode)).get();
internalCluster().fullRestart(new RestartCallback() {
@Override

View File

@ -20,6 +20,7 @@
package org.elasticsearch.rest.action.admin.cluster;
import org.elasticsearch.action.admin.cluster.configuration.AddVotingConfigExclusionsRequest;
import org.elasticsearch.common.Strings;
import org.elasticsearch.rest.RestRequest;
import org.elasticsearch.test.rest.FakeRestRequest;
import org.elasticsearch.test.rest.RestActionTestCase;
@ -50,5 +51,41 @@ public class RestAddVotingConfigExclusionActionTests extends RestActionTestCase
AddVotingConfigExclusionsRequest addVotingConfigExclusionsRequest = action.resolveVotingConfigExclusionsRequest(deprecatedRequest);
String[] expected = {"node-1","node-2", "node-3"};
assertArrayEquals(expected, addVotingConfigExclusionsRequest.getNodeDescriptions());
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeIds());
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeNames());
assertWarnings("nodeDescription is deprecated and will be removed, use nodeIds or nodeNames instead");
}
public void testResolveVotingConfigExclusionsRequestNodeIds() {
Map<String, String> params = new HashMap<>();
params.put("node_ids", "node-1,node-2,node-3");
RestRequest request = new FakeRestRequest.Builder(xContentRegistry())
.withMethod(RestRequest.Method.PUT)
.withPath("/_cluster/voting_config_exclusions")
.withParams(params)
.build();
AddVotingConfigExclusionsRequest addVotingConfigExclusionsRequest = action.resolveVotingConfigExclusionsRequest(request);
String[] expected = {"node-1","node-2", "node-3"};
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeDescriptions());
assertArrayEquals(expected, addVotingConfigExclusionsRequest.getNodeIds());
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeNames());
}
public void testResolveVotingConfigExclusionsRequestNodeNames() {
Map<String, String> params = new HashMap<>();
params.put("node_names", "node-1,node-2,node-3");
RestRequest request = new FakeRestRequest.Builder(xContentRegistry())
.withMethod(RestRequest.Method.PUT)
.withPath("/_cluster/voting_config_exclusions")
.withParams(params)
.build();
AddVotingConfigExclusionsRequest addVotingConfigExclusionsRequest = action.resolveVotingConfigExclusionsRequest(request);
String[] expected = {"node-1","node-2", "node-3"};
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeDescriptions());
assertArrayEquals(Strings.EMPTY_ARRAY, addVotingConfigExclusionsRequest.getNodeIds());
assertArrayEquals(expected, addVotingConfigExclusionsRequest.getNodeNames());
}
}

View File

@ -1855,7 +1855,7 @@ public final class InternalTestCluster extends TestCluster {
private Set<String> excludeMasters(Collection<NodeAndClient> nodeAndClients) {
assert Thread.holdsLock(this);
final Set<String> excludedNodeIds = new HashSet<>();
final Set<String> excludedNodeNames = new HashSet<>();
if (autoManageMasterNodes && nodeAndClients.size() > 0) {
final long currentMasters = nodes.values().stream().filter(NodeAndClient::isMasterEligible).count();
@ -1867,13 +1867,13 @@ public final class InternalTestCluster extends TestCluster {
// However, we do not yet have a way to be sure there's a majority left, because the voting configuration may not yet have
// been updated when the previous nodes shut down, so we must always explicitly withdraw votes.
// TODO add cluster health API to check that voting configuration is optimal so this isn't always needed
nodeAndClients.stream().filter(NodeAndClient::isMasterEligible).map(NodeAndClient::getName).forEach(excludedNodeIds::add);
assert excludedNodeIds.size() == stoppingMasters;
nodeAndClients.stream().filter(NodeAndClient::isMasterEligible).map(NodeAndClient::getName).forEach(excludedNodeNames::add);
assert excludedNodeNames.size() == stoppingMasters;
logger.info("adding voting config exclusions {} prior to restart/shutdown", excludedNodeIds);
logger.info("adding voting config exclusions {} prior to restart/shutdown", excludedNodeNames);
try {
client().execute(AddVotingConfigExclusionsAction.INSTANCE,
new AddVotingConfigExclusionsRequest(excludedNodeIds.toArray(Strings.EMPTY_ARRAY))).get();
new AddVotingConfigExclusionsRequest(excludedNodeNames.toArray(Strings.EMPTY_ARRAY))).get();
} catch (InterruptedException | ExecutionException e) {
throw new AssertionError("unexpected", e);
}
@ -1883,7 +1883,7 @@ public final class InternalTestCluster extends TestCluster {
updateMinMasterNodes(getMasterNodesCount() - Math.toIntExact(stoppingMasters));
}
}
return excludedNodeIds;
return excludedNodeNames;
}
private void removeExclusions(Set<String> excludedNodeIds) {