Enable node roles to be pluggable (#43175)

This commit introduces the possibility for a plugin to introduce
additional node roles.
This commit is contained in:
Jason Tedor 2019-06-13 14:43:14 -04:00
parent c3ce3f6891
commit 5bc3b7f741
No known key found for this signature in database
GPG Key ID: FA89F05560F16BC5
75 changed files with 835 additions and 348 deletions

View File

@ -22,6 +22,7 @@ import org.elasticsearch.Version;
import org.elasticsearch.cluster.ClusterModule;
import org.elasticsearch.cluster.EmptyClusterInfoService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.ShardRouting;
import org.elasticsearch.cluster.routing.allocation.AllocationService;
import org.elasticsearch.cluster.routing.allocation.FailedShard;
@ -92,7 +93,7 @@ public final class Allocators {
public static DiscoveryNode newNode(String nodeId, Map<String, String> attributes) {
return new DiscoveryNode("", nodeId, new TransportAddress(TransportAddress.META_ADDRESS,
portGenerator.incrementAndGet()), attributes, Sets.newHashSet(DiscoveryNode.Role.MASTER,
DiscoveryNode.Role.DATA), Version.CURRENT);
portGenerator.incrementAndGet()), attributes, Sets.newHashSet(DiscoveryNodeRole.MASTER_ROLE,
DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
}
}

View File

@ -14,7 +14,7 @@ Might look like:
[source,txt]
--------------------------------------------------
ip heap.percent ram.percent cpu load_1m load_5m load_15m node.role master name
127.0.0.1 65 99 42 3.07 mdi * mJw06l1
127.0.0.1 65 99 42 3.07 dim * mJw06l1
--------------------------------------------------
// TESTRESPONSE[s/3.07/(\\d+\\.\\d+( \\d+\\.\\d+ (\\d+\\.\\d+)?)?)?/]
// TESTRESPONSE[s/65 99 42/\\d+ \\d+ \\d+/]

View File

@ -330,7 +330,7 @@ And the response:
[source,txt]
--------------------------------------------------
ip heap.percent ram.percent cpu load_1m load_5m load_15m node.role master name
127.0.0.1 10 5 5 4.46 mdi * PB2SGZY
127.0.0.1 10 5 5 4.46 dim * PB2SGZY
--------------------------------------------------
// TESTRESPONSE[s/10 5 5 4.46/\\d+ \\d+ \\d+ (\\d+\\.\\d+)? (\\d+\\.\\d+)? (\\d+\.\\d+)?/]
// TESTRESPONSE[s/[*]/[*]/ s/PB2SGZY/.+/ non_json]

View File

@ -22,7 +22,7 @@ package org.elasticsearch.action.admin.cluster.node.info;
import org.elasticsearch.action.FailedNodeException;
import org.elasticsearch.action.support.nodes.BaseNodesResponse;
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
@ -74,8 +74,8 @@ public class NodesInfoResponse extends BaseNodesResponse<NodeInfo> implements To
}
builder.startArray("roles");
for (DiscoveryNode.Role role : nodeInfo.getNode().getRoles()) {
builder.value(role.getRoleName());
for (DiscoveryNodeRole role : nodeInfo.getNode().getRoles()) {
builder.value(role.roleName());
}
builder.endArray();

View File

@ -22,10 +22,10 @@ package org.elasticsearch.action.admin.cluster.node.stats;
import org.elasticsearch.Version;
import org.elasticsearch.action.support.nodes.BaseNodeResponse;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Nullable;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.xcontent.ToXContent.Params;
import org.elasticsearch.common.xcontent.ToXContentFragment;
import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.discovery.DiscoveryStats;
@ -277,8 +277,8 @@ public class NodeStats extends BaseNodeResponse implements ToXContentFragment {
builder.field("ip", getNode().getAddress());
builder.startArray("roles");
for (DiscoveryNode.Role role : getNode().getRoles()) {
builder.value(role.getRoleName());
for (DiscoveryNodeRole role : getNode().getRoles()) {
builder.value(role.roleName());
}
builder.endArray();

View File

@ -23,6 +23,7 @@ import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.action.TaskOperationFailure;
import org.elasticsearch.action.support.tasks.BaseTasksResponse;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.ParseField;
import org.elasticsearch.common.Strings;
@ -175,8 +176,8 @@ public class ListTasksResponse extends BaseTasksResponse implements ToXContentOb
builder.field("ip", node.getAddress());
builder.startArray("roles");
for (DiscoveryNode.Role role : node.getRoles()) {
builder.value(role.getRoleName());
for (DiscoveryNodeRole role : node.getRoles()) {
builder.value(role.roleName());
}
builder.endArray();

View File

@ -21,11 +21,11 @@ package org.elasticsearch.action.admin.cluster.stats;
import com.carrotsearch.hppc.ObjectIntHashMap;
import com.carrotsearch.hppc.cursors.ObjectIntCursor;
import org.elasticsearch.Version;
import org.elasticsearch.action.admin.cluster.node.info.NodeInfo;
import org.elasticsearch.action.admin.cluster.node.stats.NodeStats;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.collect.Tuple;
import org.elasticsearch.common.network.NetworkModule;
@ -49,6 +49,7 @@ import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.TreeMap;
import java.util.concurrent.atomic.AtomicInteger;
public class ClusterStatsNodes implements ToXContentFragment {
@ -186,27 +187,27 @@ public class ClusterStatsNodes implements ToXContentFragment {
private final int total;
private final Map<String, Integer> roles;
private Counts(List<NodeInfo> nodeInfos) {
this.roles = new HashMap<>();
for (DiscoveryNode.Role role : DiscoveryNode.Role.values()) {
this.roles.put(role.getRoleName(), 0);
private Counts(final List<NodeInfo> nodeInfos) {
// TODO: do we need to report zeros?
final Map<String, Integer> roles = new HashMap<>(DiscoveryNode.getPossibleRoleNames().size());
roles.put(COORDINATING_ONLY, 0);
for (final String possibleRoleName : DiscoveryNode.getPossibleRoleNames()) {
roles.put(possibleRoleName, 0);
}
this.roles.put(COORDINATING_ONLY, 0);
int total = 0;
for (NodeInfo nodeInfo : nodeInfos) {
for (final NodeInfo nodeInfo : nodeInfos) {
total++;
if (nodeInfo.getNode().getRoles().isEmpty()) {
Integer count = roles.get(COORDINATING_ONLY);
roles.put(COORDINATING_ONLY, ++count);
roles.merge(COORDINATING_ONLY, 1, Integer::sum);
} else {
for (DiscoveryNode.Role role : nodeInfo.getNode().getRoles()) {
Integer count = roles.get(role.getRoleName());
roles.put(role.getRoleName(), ++count);
for (DiscoveryNodeRole role : nodeInfo.getNode().getRoles()) {
roles.merge(role.roleName(), 1, Integer::sum);
}
}
}
this.total = total;
this.roles = Collections.unmodifiableMap(new HashMap<>(roles));
}
public int getTotal() {
@ -225,7 +226,7 @@ public class ClusterStatsNodes implements ToXContentFragment {
public XContentBuilder toXContent(XContentBuilder builder, Params params)
throws IOException {
builder.field(Fields.TOTAL, total);
for (Map.Entry<String, Integer> entry : roles.entrySet()) {
for (Map.Entry<String, Integer> entry : new TreeMap<>(roles).entrySet()) {
builder.field(entry.getKey(), entry.getValue());
}
return builder;

View File

@ -32,11 +32,14 @@ import org.elasticsearch.node.Node;
import java.io.IOException;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.function.Function;
import java.util.function.Predicate;
import java.util.stream.Collectors;
/**
@ -78,8 +81,7 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
private final TransportAddress address;
private final Map<String, String> attributes;
private final Version version;
private final Set<Role> roles;
private final Set<DiscoveryNodeRole> roles;
/**
* Creates a new {@link DiscoveryNode}
@ -95,7 +97,7 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
* @param version the version of the node
*/
public DiscoveryNode(final String id, TransportAddress address, Version version) {
this(id, address, Collections.emptyMap(), EnumSet.allOf(Role.class), version);
this(id, address, Collections.emptyMap(), DiscoveryNodeRole.BUILT_IN_ROLES, version);
}
/**
@ -113,7 +115,7 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
* @param roles node roles
* @param version the version of the node
*/
public DiscoveryNode(String id, TransportAddress address, Map<String, String> attributes, Set<Role> roles,
public DiscoveryNode(String id, TransportAddress address, Map<String, String> attributes, Set<DiscoveryNodeRole> roles,
Version version) {
this("", id, address, attributes, roles, version);
}
@ -135,7 +137,7 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
* @param version the version of the node
*/
public DiscoveryNode(String nodeName, String nodeId, TransportAddress address,
Map<String, String> attributes, Set<Role> roles, Version version) {
Map<String, String> attributes, Set<DiscoveryNodeRole> roles, Version version) {
this(nodeName, nodeId, UUIDs.randomBase64UUID(), address.address().getHostString(), address.getAddress(), address, attributes,
roles, version);
}
@ -159,7 +161,7 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
* @param version the version of the node
*/
public DiscoveryNode(String nodeName, String nodeId, String ephemeralId, String hostName, String hostAddress,
TransportAddress address, Map<String, String> attributes, Set<Role> roles, Version version) {
TransportAddress address, Map<String, String> attributes, Set<DiscoveryNodeRole> roles, Version version) {
if (nodeName != null) {
this.nodeName = nodeName.intern();
} else {
@ -178,37 +180,28 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
this.attributes = Collections.unmodifiableMap(attributes);
//verify that no node roles are being provided as attributes
Predicate<Map<String, String>> predicate = (attrs) -> {
for (Role role : Role.values()) {
assert attrs.containsKey(role.getRoleName()) == false;
boolean success = true;
for (final DiscoveryNodeRole role : DiscoveryNode.roleNameToPossibleRoles.values()) {
success &= attrs.containsKey(role.roleName()) == false;
assert success : role.roleName();
}
return true;
return success;
};
assert predicate.test(attributes);
Set<Role> rolesSet = EnumSet.noneOf(Role.class);
rolesSet.addAll(roles);
this.roles = Collections.unmodifiableSet(rolesSet);
assert predicate.test(attributes) : attributes;
this.roles = Collections.unmodifiableSet(new HashSet<>(roles));
}
/** Creates a DiscoveryNode representing the local node. */
public static DiscoveryNode createLocal(Settings settings, TransportAddress publishAddress, String nodeId) {
Map<String, String> attributes = Node.NODE_ATTRIBUTES.getAsMap(settings);
Set<Role> roles = getRolesFromSettings(settings);
Set<DiscoveryNodeRole> roles = getRolesFromSettings(settings);
return new DiscoveryNode(Node.NODE_NAME_SETTING.get(settings), nodeId, publishAddress, attributes, roles, Version.CURRENT);
}
/** extract node roles from the given settings */
public static Set<Role> getRolesFromSettings(Settings settings) {
Set<Role> roles = EnumSet.noneOf(Role.class);
if (Node.NODE_INGEST_SETTING.get(settings)) {
roles.add(Role.INGEST);
}
if (Node.NODE_MASTER_SETTING.get(settings)) {
roles.add(Role.MASTER);
}
if (Node.NODE_DATA_SETTING.get(settings)) {
roles.add(Role.DATA);
}
return roles;
public static Set<DiscoveryNodeRole> getRolesFromSettings(final Settings settings) {
return Collections.unmodifiableSet(
roleNameToPossibleRoles.values().stream().filter(s -> s.roleSetting().get(settings)).collect(Collectors.toSet()));
}
/**
@ -229,10 +222,41 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
this.attributes.put(in.readString(), in.readString());
}
int rolesSize = in.readVInt();
this.roles = EnumSet.noneOf(Role.class);
for (int i = 0; i < rolesSize; i++) {
this.roles.add(in.readEnum(Role.class));
final Set<DiscoveryNodeRole> roles = new HashSet<>(rolesSize);
if (in.getVersion().onOrAfter(Version.V_7_3_0)) {
for (int i = 0; i < rolesSize; i++) {
final String roleName = in.readString();
final String roleNameAbbreviation = in.readString();
final DiscoveryNodeRole role = roleNameToPossibleRoles.get(roleName);
if (role == null) {
roles.add(new DiscoveryNodeRole.UnknownRole(roleName, roleNameAbbreviation));
} else {
assert roleName.equals(role.roleName()) : "role name [" + roleName + "] does not match role [" + role.roleName() + "]";
assert roleNameAbbreviation.equals(role.roleNameAbbreviation())
: "role name abbreviation [" + roleName + "] does not match role [" + role.roleNameAbbreviation() + "]";
roles.add(role);
}
}
} else {
// an old node will only send us legacy roles since pluggable roles is a new concept
for (int i = 0; i < rolesSize; i++) {
final LegacyRole legacyRole = in.readEnum(LegacyRole.class);
switch (legacyRole) {
case MASTER:
roles.add(DiscoveryNodeRole.MASTER_ROLE);
break;
case DATA:
roles.add(DiscoveryNodeRole.DATA_ROLE);
break;
case INGEST:
roles.add(DiscoveryNodeRole.INGEST_ROLE);
break;
default:
throw new AssertionError(legacyRole.roleName());
}
}
}
this.roles = Collections.unmodifiableSet(new HashSet<>(roles));
this.version = Version.readVersion(in);
}
@ -249,9 +273,26 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
out.writeString(entry.getKey());
out.writeString(entry.getValue());
}
out.writeVInt(roles.size());
for (Role role : roles) {
out.writeEnum(role);
if (out.getVersion().onOrAfter(Version.V_7_3_0)) {
out.writeVInt(roles.size());
for (final DiscoveryNodeRole role : roles) {
out.writeString(role.roleName());
out.writeString(role.roleNameAbbreviation());
}
} else {
// an old node will only understand legacy roles since pluggable roles is a new concept
final List<DiscoveryNodeRole> rolesToWrite =
roles.stream().filter(DiscoveryNodeRole.BUILT_IN_ROLES::contains).collect(Collectors.toList());
out.writeVInt(rolesToWrite.size());
for (final DiscoveryNodeRole role : rolesToWrite) {
if (role == DiscoveryNodeRole.MASTER_ROLE) {
out.writeEnum(LegacyRole.MASTER);
} else if (role == DiscoveryNodeRole.DATA_ROLE) {
out.writeEnum(LegacyRole.DATA);
} else if (role == DiscoveryNodeRole.INGEST_ROLE) {
out.writeEnum(LegacyRole.INGEST);
}
}
}
Version.writeVersion(version, out);
}
@ -299,28 +340,28 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
* Should this node hold data (shards) or not.
*/
public boolean isDataNode() {
return roles.contains(Role.DATA);
return roles.contains(DiscoveryNodeRole.DATA_ROLE);
}
/**
* Can this node become master or not.
*/
public boolean isMasterNode() {
return roles.contains(Role.MASTER);
return roles.contains(DiscoveryNodeRole.MASTER_ROLE);
}
/**
* Returns a boolean that tells whether this an ingest node or not
*/
public boolean isIngestNode() {
return roles.contains(Role.INGEST);
return roles.contains(DiscoveryNodeRole.INGEST_ROLE);
}
/**
* Returns a set of all the roles that the node fulfills.
* If the node doesn't have any specific role, the set is returned empty, which means that the node is a coordinating only node.
*/
public Set<Role> getRoles() {
public Set<DiscoveryNodeRole> getRoles() {
return roles;
}
@ -391,29 +432,44 @@ public class DiscoveryNode implements Writeable, ToXContentFragment {
return builder;
}
private static Map<String, DiscoveryNodeRole> roleNameToPossibleRoles;
public static void setPossibleRoles(final Set<DiscoveryNodeRole> possibleRoles) {
final Map<String, DiscoveryNodeRole> roleNameToPossibleRoles = Collections.unmodifiableMap(
possibleRoles.stream().collect(Collectors.toMap(DiscoveryNodeRole::roleName, Function.identity())));
// collect the abbreviation names into a map to ensure that there are not any duplicate abbreviations
final Map<String, DiscoveryNodeRole> roleNameAbbreviationToPossibleRoles = Collections.unmodifiableMap(
roleNameToPossibleRoles.values()
.stream()
.collect(Collectors.toMap(DiscoveryNodeRole::roleNameAbbreviation, Function.identity())));
assert roleNameToPossibleRoles.size() == roleNameAbbreviationToPossibleRoles.size() :
"roles by name [" + roleNameToPossibleRoles + "], roles by name abbreviation [" + roleNameAbbreviationToPossibleRoles + "]";
DiscoveryNode.roleNameToPossibleRoles = roleNameToPossibleRoles;
}
public static Set<String> getPossibleRoleNames() {
return roleNameToPossibleRoles.keySet();
}
/**
* Enum that holds all the possible roles that that a node can fulfill in a cluster.
* Each role has its name and a corresponding abbreviation used by cat apis.
*/
public enum Role {
MASTER("master", "m"),
DATA("data", "d"),
INGEST("ingest", "i");
private enum LegacyRole {
MASTER("master"),
DATA("data"),
INGEST("ingest");
private final String roleName;
private final String abbreviation;
Role(String roleName, String abbreviation) {
LegacyRole(final String roleName) {
this.roleName = roleName;
this.abbreviation = abbreviation;
}
public String getRoleName() {
public String roleName() {
return roleName;
}
public String getAbbreviation() {
return abbreviation;
}
}
}

View File

@ -0,0 +1,141 @@
/*
* Licensed to Elasticsearch under one or more contributor
* license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright
* ownership. Elasticsearch 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.elasticsearch.cluster.node;
import org.elasticsearch.common.settings.Setting;
import org.elasticsearch.node.Node;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.Objects;
import java.util.Set;
/**
* Represents a node role.
*/
public abstract class DiscoveryNodeRole {
private final String roleName;
/**
* The name of the role.
*
* @return the role name
*/
public final String roleName() {
return roleName;
}
private final String roleNameAbbreviation;
/**
* The abbreviation of the name of the role. This is used in the cat nodes API to display an abbreviated version of the name of the
* role.
*
* @return the role name abbreviation
*/
public final String roleNameAbbreviation() {
return roleNameAbbreviation;
}
protected DiscoveryNodeRole(final String roleName, final String roleNameAbbreviation) {
this.roleName = Objects.requireNonNull(roleName);
this.roleNameAbbreviation = Objects.requireNonNull(roleNameAbbreviation);
}
protected abstract Setting<Boolean> roleSetting();
@Override
public String toString() {
return "DiscoveryNodeRole{" +
"roleName='" + roleName + '\'' +
", roleNameAbbreviation='" + roleNameAbbreviation + '\'' +
'}';
}
/**
* Represents the role for a data node.
*/
public static final DiscoveryNodeRole DATA_ROLE = new DiscoveryNodeRole("data", "d") {
@Override
protected Setting<Boolean> roleSetting() {
return Node.NODE_DATA_SETTING;
}
};
/**
* Represents the role for an ingest node.
*/
public static final DiscoveryNodeRole INGEST_ROLE = new DiscoveryNodeRole("ingest", "i") {
@Override
protected Setting<Boolean> roleSetting() {
return Node.NODE_INGEST_SETTING;
}
};
/**
* Represents the role for a master-eligible node.
*/
public static final DiscoveryNodeRole MASTER_ROLE = new DiscoveryNodeRole("master", "m") {
@Override
protected Setting<Boolean> roleSetting() {
return Node.NODE_MASTER_SETTING;
}
};
/**
* The built-in node roles.
*/
public static Set<DiscoveryNodeRole> BUILT_IN_ROLES =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DATA_ROLE, INGEST_ROLE, MASTER_ROLE)));
/**
* Represents an unknown role. This can occur if a newer version adds a role that an older version does not know about, or a newer
* version removes a role that an older version knows about.
*/
static class UnknownRole extends DiscoveryNodeRole {
/**
* Construct an unknown role with the specified role name and role name abbreviation.
*
* @param roleName the role name
* @param roleNameAbbreviation the role name abbreviation
*/
UnknownRole(final String roleName, final String roleNameAbbreviation) {
super(roleName, roleNameAbbreviation);
}
@Override
protected Setting<Boolean> roleSetting() {
// since this setting is not registered, it will always return false when testing if the local node has the role
assert false;
return Setting.boolSetting("node. " + roleName(), false, Setting.Property.NodeScope);
}
}
}

View File

@ -353,19 +353,19 @@ public class DiscoveryNodes extends AbstractDiffable<DiscoveryNodes> implements
if (index != -1) {
String matchAttrName = nodeId.substring(0, index);
String matchAttrValue = nodeId.substring(index + 1);
if (DiscoveryNode.Role.DATA.getRoleName().equals(matchAttrName)) {
if (DiscoveryNodeRole.DATA_ROLE.roleName().equals(matchAttrName)) {
if (Booleans.parseBoolean(matchAttrValue, true)) {
resolvedNodesIds.addAll(dataNodes.keys());
} else {
resolvedNodesIds.removeAll(dataNodes.keys());
}
} else if (DiscoveryNode.Role.MASTER.getRoleName().equals(matchAttrName)) {
} else if (DiscoveryNodeRole.MASTER_ROLE.roleName().equals(matchAttrName)) {
if (Booleans.parseBoolean(matchAttrValue, true)) {
resolvedNodesIds.addAll(masterNodes.keys());
} else {
resolvedNodesIds.removeAll(masterNodes.keys());
}
} else if (DiscoveryNode.Role.INGEST.getRoleName().equals(matchAttrName)) {
} else if (DiscoveryNodeRole.INGEST_ROLE.roleName().equals(matchAttrName)) {
if (Booleans.parseBoolean(matchAttrValue, true)) {
resolvedNodesIds.addAll(ingestNodes.keys());
} else {

View File

@ -55,6 +55,7 @@ import org.elasticsearch.cluster.metadata.MetaDataCreateIndexService;
import org.elasticsearch.cluster.metadata.MetaDataIndexUpgradeService;
import org.elasticsearch.cluster.metadata.TemplateUpgradeService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.RoutingService;
import org.elasticsearch.cluster.routing.allocation.DiskThresholdMonitor;
import org.elasticsearch.cluster.service.ClusterService;
@ -156,6 +157,7 @@ import org.elasticsearch.usage.UsageService;
import org.elasticsearch.watcher.ResourceWatcherService;
import javax.net.ssl.SNIHostName;
import java.io.BufferedWriter;
import java.io.Closeable;
import java.io.IOException;
@ -308,6 +310,14 @@ public class Node implements Closeable {
this.pluginsService = new PluginsService(tmpSettings, environment.configFile(), environment.modulesFile(),
environment.pluginsFile(), classpathPlugins);
final Settings settings = pluginsService.updatedSettings();
final Set<DiscoveryNodeRole> possibleRoles = Stream.concat(
DiscoveryNodeRole.BUILT_IN_ROLES.stream(),
pluginsService.filterPlugins(Plugin.class)
.stream()
.map(Plugin::getRoles)
.flatMap(Set::stream))
.collect(Collectors.toSet());
DiscoveryNode.setPossibleRoles(possibleRoles);
localNodeFactory = new LocalNodeFactory(settings, nodeEnvironment.nodeId());
// create the environment based on the finalized (processed) view of the settings

View File

@ -25,6 +25,7 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.IndexTemplateMetaData;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.component.LifecycleComponent;
import org.elasticsearch.common.inject.Module;
@ -50,6 +51,7 @@ import java.util.Collections;
import java.util.List;
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import java.util.function.UnaryOperator;
/**
@ -236,6 +238,10 @@ public abstract class Plugin implements Closeable {
*/
public List<BootstrapCheck> getBootstrapChecks() { return Collections.emptyList(); }
public Set<DiscoveryNodeRole> getRoles() {
return Collections.emptySet();
}
/**
* Close the resources opened by this plugin.
*

View File

@ -29,6 +29,7 @@ import org.elasticsearch.action.admin.cluster.state.ClusterStateRequest;
import org.elasticsearch.action.admin.cluster.state.ClusterStateResponse;
import org.elasticsearch.client.node.NodeClient;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.Table;
@ -324,7 +325,7 @@ public class RestNodesAction extends AbstractCatAction {
if (node.getRoles().isEmpty()) {
roles = "-";
} else {
roles = node.getRoles().stream().map(DiscoveryNode.Role::getAbbreviation).collect(Collectors.joining());
roles = node.getRoles().stream().map(DiscoveryNodeRole::roleNameAbbreviation).sorted().collect(Collectors.joining());
}
table.addCell(roles);
table.addCell(masterId == null ? "x" : masterId.equals(node.getId()) ? "*" : "-");

View File

@ -22,6 +22,7 @@ package org.elasticsearch.transport;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.metadata.ClusterNameExpressionResolver;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.UUIDs;
import org.elasticsearch.common.collect.Tuple;
@ -38,7 +39,6 @@ import java.net.UnknownHostException;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.List;
import java.util.Locale;
@ -226,7 +226,7 @@ public abstract class RemoteClusterAware {
TransportAddress transportAddress = new TransportAddress(TransportAddress.META_ADDRESS, 0);
String hostName = address.substring(0, indexOfPortSeparator(address));
return new DiscoveryNode("", clusterName + "#" + address, UUIDs.randomBase64UUID(), hostName, address,
transportAddress, Collections.singletonMap("server_name", hostName), EnumSet.allOf(DiscoveryNode.Role.class),
transportAddress, Collections.singletonMap("server_name", hostName), DiscoveryNodeRole.BUILT_IN_ROLES,
Version.CURRENT.minimumCompatibilityVersion());
} else {
TransportAddress transportAddress = new TransportAddress(RemoteClusterAware.parseSeedAddress(address));

View File

@ -25,16 +25,16 @@ import org.elasticsearch.cluster.coordination.CoordinationMetaData;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfigExclusion;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes.Builder;
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;
@ -55,14 +55,29 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
}
public void testResolve() {
final DiscoveryNode localNode
= new DiscoveryNode("local", "local", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode localNode = new DiscoveryNode(
"local",
"local",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion localNodeExclusion = new VotingConfigExclusion(localNode);
final DiscoveryNode otherNode1
= new DiscoveryNode("other1", "other1", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode otherNode1 = new DiscoveryNode(
"other1",
"other1",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode1Exclusion = new VotingConfigExclusion(otherNode1);
final DiscoveryNode otherNode2
= new DiscoveryNode("other2", "other2", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode otherNode2 = new DiscoveryNode(
"other2",
"other2",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode2Exclusion = new VotingConfigExclusion(otherNode2);
final DiscoveryNode otherDataNode
= new DiscoveryNode("data", "data", buildNewFakeTransportAddress(), emptyMap(), emptySet(), Version.CURRENT);
@ -85,14 +100,29 @@ public class AddVotingConfigExclusionsRequestTests extends ESTestCase {
}
public void testResolveAndCheckMaximum() {
final DiscoveryNode localNode
= new DiscoveryNode("local", "local", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode localNode = new DiscoveryNode(
"local",
"local",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion localNodeExclusion = new VotingConfigExclusion(localNode);
final DiscoveryNode otherNode1
= new DiscoveryNode("other1", "other1", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode otherNode1 = new DiscoveryNode(
"other1",
"other1",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode1Exclusion = new VotingConfigExclusion(otherNode1);
final DiscoveryNode otherNode2
= new DiscoveryNode("other2", "other2", buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
final DiscoveryNode otherNode2 = new DiscoveryNode(
"other2",
"other2",
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
final VotingConfigExclusion otherNode2Exclusion = new VotingConfigExclusion(otherNode2);
final ClusterState.Builder builder = ClusterState.builder(new ClusterName("cluster")).nodes(new Builder()

View File

@ -28,12 +28,12 @@ import org.elasticsearch.cluster.ClusterStateObserver;
import org.elasticsearch.cluster.ClusterStateObserver.Listener;
import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.coordination.CoordinationMetaData;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfigExclusion;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
import org.elasticsearch.cluster.metadata.IndexNameExpressionResolver;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes.Builder;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.StreamInput;
@ -52,6 +52,7 @@ 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;
@ -60,7 +61,6 @@ 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;
@ -96,7 +96,13 @@ public class TransportAddVotingConfigExclusionsActionTests extends ESTestCase {
}
private static DiscoveryNode makeDiscoveryNode(String name) {
return new DiscoveryNode(name, name, buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER), Version.CURRENT);
return new DiscoveryNode(
name,
name,
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
}
@AfterClass

View File

@ -25,7 +25,7 @@ import org.elasticsearch.action.admin.cluster.node.stats.NodeStats;
import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsResponse;
import org.elasticsearch.client.Requests;
import org.elasticsearch.cluster.health.ClusterHealthStatus;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Priority;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.util.concurrent.EsExecutors;
@ -63,9 +63,9 @@ public class ClusterStatsIT extends ESIntegTestCase {
int total = 1;
internalCluster().startNode();
Map<String, Integer> expectedCounts = new HashMap<>();
expectedCounts.put(DiscoveryNode.Role.DATA.getRoleName(), 1);
expectedCounts.put(DiscoveryNode.Role.MASTER.getRoleName(), 1);
expectedCounts.put(DiscoveryNode.Role.INGEST.getRoleName(), 1);
expectedCounts.put(DiscoveryNodeRole.DATA_ROLE.roleName(), 1);
expectedCounts.put(DiscoveryNodeRole.MASTER_ROLE.roleName(), 1);
expectedCounts.put(DiscoveryNodeRole.INGEST_ROLE.roleName(), 1);
expectedCounts.put(ClusterStatsNodes.Counts.COORDINATING_ONLY, 0);
int numNodes = randomIntBetween(1, 5);
@ -84,13 +84,13 @@ public class ClusterStatsIT extends ESIntegTestCase {
waitForNodes(total);
if (isDataNode) {
incrementCountForRole(DiscoveryNode.Role.DATA.getRoleName(), expectedCounts);
incrementCountForRole(DiscoveryNodeRole.DATA_ROLE.roleName(), expectedCounts);
}
if (isMasterNode) {
incrementCountForRole(DiscoveryNode.Role.MASTER.getRoleName(), expectedCounts);
incrementCountForRole(DiscoveryNodeRole.MASTER_ROLE.roleName(), expectedCounts);
}
if (isIngestNode) {
incrementCountForRole(DiscoveryNode.Role.INGEST.getRoleName(), expectedCounts);
incrementCountForRole(DiscoveryNodeRole.INGEST_ROLE.roleName(), expectedCounts);
}
if (!isDataNode && !isMasterNode && !isIngestNode) {
incrementCountForRole(ClusterStatsNodes.Counts.COORDINATING_ONLY, expectedCounts);

View File

@ -30,6 +30,7 @@ import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.routing.ShardRoutingState;
@ -45,6 +46,7 @@ import org.elasticsearch.test.gateway.TestGatewayAllocator;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.Set;
import static java.util.Collections.emptyMap;
import static org.hamcrest.Matchers.equalTo;
@ -216,7 +218,8 @@ public class TransportResizeActionTests extends ESTestCase {
}
private DiscoveryNode newNode(String nodeId) {
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(),
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.DATA))), Version.CURRENT);
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE)));
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
}
}

View File

@ -28,6 +28,7 @@ import org.elasticsearch.client.node.NodeClient;
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.Randomness;
@ -154,12 +155,12 @@ public class TransportMultiSearchActionTests extends ESTestCase {
DiscoveryNodes.Builder builder = DiscoveryNodes.builder();
for (int i = 0; i < numDataNodes; i++) {
builder.add(new DiscoveryNode("_id" + i, buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.DATA), Version.CURRENT));
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT));
}
builder.add(new DiscoveryNode("master", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.MASTER), Version.CURRENT));
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT));
builder.add(new DiscoveryNode("ingest", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.INGEST), Version.CURRENT));
Collections.singleton(DiscoveryNodeRole.INGEST_ROLE), Version.CURRENT));
ClusterState state = ClusterState.builder(new ClusterName("_name")).nodes(builder).build();
int result = TransportMultiSearchAction.defaultMaxConcurrentSearches(10, state);

View File

@ -34,14 +34,15 @@ import org.elasticsearch.cluster.block.ClusterBlock;
import org.elasticsearch.cluster.block.ClusterBlockException;
import org.elasticsearch.cluster.block.ClusterBlockLevel;
import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.coordination.FailedToCommitClusterStateException;
import org.elasticsearch.cluster.metadata.IndexNameExpressionResolver;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.unit.TimeValue;
import org.elasticsearch.cluster.coordination.FailedToCommitClusterStateException;
import org.elasticsearch.discovery.MasterNotDiscoveredException;
import org.elasticsearch.rest.RestStatus;
import org.elasticsearch.tasks.Task;
@ -95,9 +96,9 @@ public class TransportMasterNodeActionTests extends ESTestCase {
transportService.start();
transportService.acceptIncomingRequests();
localNode = new DiscoveryNode("local_node", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.MASTER), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
remoteNode = new DiscoveryNode("remote_node", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.MASTER), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
allNodes = new DiscoveryNode[]{localNode, remoteNode};
}

View File

@ -27,6 +27,7 @@ import org.elasticsearch.action.support.broadcast.node.TransportBroadcastByNodeA
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.StreamInput;
@ -43,7 +44,6 @@ import org.junit.BeforeClass;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
@ -188,7 +188,7 @@ public class TransportNodesActionTests extends ESTestCase {
List<DiscoveryNode> discoveryNodes = new ArrayList<>();
for (int i = 0; i < numNodes; i++) {
Map<String, String> attributes = new HashMap<>();
Set<DiscoveryNode.Role> roles = new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values())));
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
if (frequently()) {
attributes.put("custom", randomBoolean() ? "match" : randomAlphaOfLengthBetween(3, 5));
}
@ -235,7 +235,7 @@ public class TransportNodesActionTests extends ESTestCase {
);
}
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNode.Role> roles) {
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNodeRole> roles) {
String node = "node_" + nodeId;
return new DiscoveryNode(node, node, buildNewFakeTransportAddress(), attributes, roles, Version.CURRENT);
}

View File

@ -26,6 +26,7 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.AllocationId;
import org.elasticsearch.cluster.routing.IndexRoutingTable;
@ -41,7 +42,6 @@ import org.elasticsearch.index.shard.ShardId;
import org.elasticsearch.test.ESTestCase;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
@ -407,7 +407,7 @@ public class ClusterStateCreationUtils {
private static DiscoveryNode newNode(int nodeId) {
return new DiscoveryNode("node_" + nodeId, ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
}
private static String selectAndRemove(Set<String> strings) {

View File

@ -33,6 +33,7 @@ import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.IndexNameExpressionResolver;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -64,7 +65,6 @@ import org.junit.Before;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.HashSet;
import java.util.List;
import java.util.Objects;
@ -128,7 +128,7 @@ public class TransportReplicationAllPermitsAcquisitionTests extends IndexShardTe
clusterService = createClusterService(threadPool);
final ClusterState.Builder state = ClusterState.builder(clusterService.state());
Set<DiscoveryNode.Role> roles = new HashSet<>(Arrays.asList(DiscoveryNode.Role.values()));
Set<DiscoveryNodeRole> roles = new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES);
DiscoveryNode node1 = new DiscoveryNode("_name1", "_node1", buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
DiscoveryNode node2 = new DiscoveryNode("_name2", "_node2", buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
state.nodes(DiscoveryNodes.builder()

View File

@ -30,6 +30,7 @@ import org.elasticsearch.client.Client;
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.io.stream.NamedWriteableRegistry;
import org.elasticsearch.common.network.NetworkModule;
@ -173,7 +174,7 @@ public class TransportClientHeadersTests extends AbstractClientHeadersTestCase {
//the sniffer detects only data nodes
builder.nodes(DiscoveryNodes.builder().add(new DiscoveryNode("node_id", "someId", "some_ephemeralId_id",
address.address().getHostString(), address.getAddress(), address, Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.DATA), Version.CURRENT)));
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT)));
((TransportResponseHandler<ClusterStateResponse>) handler)
.handleResponse(new ClusterStateResponse(cluster1, builder.build(), false));
clusterStateLatch.countDown();

View File

@ -27,6 +27,7 @@ import org.elasticsearch.cluster.metadata.IndexGraveyard;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RoutingTable;
import org.elasticsearch.common.UUIDs;
@ -422,24 +423,24 @@ public class ClusterChangedEventTests extends ESTestCase {
final int localNodeIndex = isLocalMaster ? 0 : randomIntBetween(1, numNodes - 1); // randomly assign the local node if not master
for (int i = 0; i < numNodes; i++) {
final String nodeId = NODE_ID_PREFIX + i;
Set<DiscoveryNode.Role> roles = new HashSet<>();
Set<DiscoveryNodeRole> roles = new HashSet<>();
if (i == 0) {
// the master node
builder.masterNodeId(nodeId);
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
} else if (i == 1) {
// the alternate master node
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
} else if (i == 2) {
// we need at least one data node
roles.add(DiscoveryNode.Role.DATA);
roles.add(DiscoveryNodeRole.DATA_ROLE);
} else {
// remaining nodes can be anything (except for master)
if (randomBoolean()) {
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
}
if (randomBoolean()) {
roles.add(DiscoveryNode.Role.DATA);
roles.add(DiscoveryNodeRole.DATA_ROLE);
}
}
final DiscoveryNode node = newNode(nodeId, roles);
@ -452,7 +453,7 @@ public class ClusterChangedEventTests extends ESTestCase {
}
// Create a new DiscoveryNode
private static DiscoveryNode newNode(final String nodeId, Set<DiscoveryNode.Role> roles) {
private static DiscoveryNode newNode(final String nodeId, Set<DiscoveryNodeRole> roles) {
return new DiscoveryNode(nodeId, nodeId, nodeId, "host", "host_address", buildNewFakeTransportAddress(),
Collections.emptyMap(), roles, Version.CURRENT);
}

View File

@ -25,6 +25,7 @@ import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.support.PlainActionFuture;
import org.elasticsearch.cluster.coordination.DeterministicTaskQueue;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.CheckedRunnable;
import org.elasticsearch.common.UUIDs;
@ -51,7 +52,6 @@ import org.junit.After;
import org.junit.Before;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
@ -80,7 +80,7 @@ public class NodeConnectionsServiceTests extends ESTestCase {
private List<DiscoveryNode> generateNodes() {
List<DiscoveryNode> nodes = new ArrayList<>();
for (int i = randomIntBetween(20, 50); i > 0; i--) {
Set<DiscoveryNode.Role> roles = new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values())));
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
nodes.add(new DiscoveryNode("node_" + i, "" + i, buildNewFakeTransportAddress(), Collections.emptyMap(),
roles, Version.CURRENT));
}

View File

@ -21,7 +21,7 @@ package org.elasticsearch.cluster.coordination;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.discovery.DiscoveryModule;
import org.elasticsearch.node.Node;
@ -43,7 +43,6 @@ import java.util.stream.Stream;
import static java.util.Collections.emptyList;
import static java.util.Collections.emptyMap;
import static java.util.Collections.emptySet;
import static java.util.Collections.singleton;
import static java.util.Collections.singletonList;
import static java.util.Collections.singletonMap;
import static org.elasticsearch.cluster.coordination.ClusterBootstrapService.BOOTSTRAP_PLACEHOLDER_PREFIX;
@ -89,8 +88,13 @@ public class ClusterBootstrapServiceTests extends ESTestCase {
}
private DiscoveryNode newDiscoveryNode(String nodeName) {
return new DiscoveryNode(nodeName, randomAlphaOfLength(10), buildNewFakeTransportAddress(), emptyMap(), singleton(Role.MASTER),
Version.CURRENT);
return new DiscoveryNode(
nodeName,
randomAlphaOfLength(10),
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
}
public void testBootstrapsAutomaticallyWithDefaultConfiguration() {
@ -157,7 +161,7 @@ public class ClusterBootstrapServiceTests extends ESTestCase {
public void testDoesNothingByDefaultIfZen1NodesDiscovered() {
final DiscoveryNode zen1Node = new DiscoveryNode("zen1", buildNewFakeTransportAddress(), singletonMap("zen1", "true"),
singleton(Role.MASTER), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
ClusterBootstrapService clusterBootstrapService = new ClusterBootstrapService(Settings.EMPTY, transportService, () ->
Stream.of(localNode, zen1Node).collect(Collectors.toSet()), () -> false, vc -> {
throw new AssertionError("should not be called");
@ -356,7 +360,7 @@ public class ClusterBootstrapServiceTests extends ESTestCase {
public void testDoesNotBootstrapsIfZen1NodesDiscovered() {
final DiscoveryNode zen1Node = new DiscoveryNode("zen1", buildNewFakeTransportAddress(), singletonMap("zen1", "true"),
singleton(Role.MASTER), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
ClusterBootstrapService clusterBootstrapService = new ClusterBootstrapService(Settings.builder().putList(
INITIAL_MASTER_NODES_SETTING.getKey(), localNode.getName(), otherNode1.getName(), otherNode2.getName()).build(),
@ -420,10 +424,22 @@ public class ClusterBootstrapServiceTests extends ESTestCase {
clusterBootstrapService.onFoundPeersUpdated();
deterministicTaskQueue.runAllTasks();
discoveredNodes.set(Stream.of(new DiscoveryNode(otherNode1.getName(), randomAlphaOfLength(10), buildNewFakeTransportAddress(),
emptyMap(), singleton(Role.MASTER), Version.CURRENT),
new DiscoveryNode("yet-another-node", randomAlphaOfLength(10), otherNode1.getAddress(), emptyMap(), singleton(Role.MASTER),
Version.CURRENT)).collect(Collectors.toList()));
discoveredNodes.set(Stream.of(
new DiscoveryNode(
otherNode1.getName(),
randomAlphaOfLength(10),
buildNewFakeTransportAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT),
new DiscoveryNode(
"yet-another-node",
randomAlphaOfLength(10),
otherNode1.getAddress(),
emptyMap(),
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT))
.collect(Collectors.toList()));
clusterBootstrapService.onFoundPeersUpdated();
deterministicTaskQueue.runAllTasks();

View File

@ -23,13 +23,13 @@ import org.elasticsearch.Version;
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
import org.elasticsearch.cluster.coordination.CoordinationState.PersistedState;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.UUIDs;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.cluster.coordination.CoordinationState.PersistedState;
import org.elasticsearch.common.transport.TransportAddress;
import org.elasticsearch.common.util.set.Sets;
import org.elasticsearch.node.Node;
@ -39,7 +39,6 @@ import org.junit.Before;
import java.util.ArrayList;
import java.util.Collections;
import java.util.EnumSet;
import java.util.List;
import java.util.Optional;
import java.util.Set;
@ -89,7 +88,7 @@ public class CoordinationStateTests extends ESTestCase {
return new DiscoveryNode("", id,
UUIDs.randomBase64UUID(random()), // generated deterministically for repeatable tests
address.address().getHostString(), address.getAddress(), address, Collections.emptyMap(),
EnumSet.allOf(Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
}
public void testSetInitialState() {

View File

@ -44,7 +44,7 @@ import org.elasticsearch.cluster.coordination.LinearizabilityChecker.SequentialS
import org.elasticsearch.cluster.metadata.Manifest;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.allocation.AllocationService;
import org.elasticsearch.cluster.service.ClusterApplierService;
import org.elasticsearch.cluster.service.ClusterService;
@ -92,7 +92,6 @@ import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
@ -1960,7 +1959,7 @@ public class CoordinatorTests extends ESTestCase {
final DiscoveryNode newLocalNode = new DiscoveryNode(localNode.getName(), localNode.getId(),
UUIDs.randomBase64UUID(random()), // generated deterministically for repeatable tests
address.address().getHostString(), address.getAddress(), address, Collections.emptyMap(),
localNode.isMasterNode() ? EnumSet.allOf(Role.class) : emptySet(), Version.CURRENT);
localNode.isMasterNode() ? DiscoveryNodeRole.BUILT_IN_ROLES : emptySet(), Version.CURRENT);
return new ClusterNode(nodeIndex, newLocalNode,
node -> new MockPersistedState(newLocalNode, persistedState, adaptGlobalMetaData, adaptCurrentTerm), nodeSettings);
}
@ -2323,7 +2322,7 @@ public class CoordinatorTests extends ESTestCase {
return new DiscoveryNode("", "node" + nodeIndex,
UUIDs.randomBase64UUID(random()), // generated deterministically for repeatable tests
address.address().getHostString(), address.getAddress(), address, Collections.emptyMap(),
masterEligible ? EnumSet.allOf(Role.class) : emptySet(), Version.CURRENT);
masterEligible ? DiscoveryNodeRole.BUILT_IN_ROLES : emptySet(), Version.CURRENT);
}
/**

View File

@ -24,6 +24,7 @@ import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.coordination.Coordinator.Mode;
import org.elasticsearch.cluster.coordination.FollowersChecker.FollowerCheckRequest;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.settings.Settings;
@ -43,7 +44,6 @@ import org.elasticsearch.transport.TransportResponseHandler;
import org.elasticsearch.transport.TransportService;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.Comparator;
import java.util.HashMap;
@ -577,13 +577,13 @@ public class FollowersCheckerTests extends ESTestCase {
attributes.put("custom", randomBoolean() ? "match" : randomAlphaOfLengthBetween(3, 5));
}
final DiscoveryNode node = newNode(i, attributes,
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))));
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)));
nodesList.add(node);
}
return nodesList;
}
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNode.Role> roles) {
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNodeRole> roles) {
return new DiscoveryNode("name_" + nodeId, "node_" + nodeId, buildNewFakeTransportAddress(), attributes, roles,
Version.CURRENT);
}

View File

@ -22,11 +22,12 @@ import org.apache.logging.log4j.message.ParameterizedMessage;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
import org.elasticsearch.cluster.ESAllocationTestCase;
import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
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.cluster.service.MasterService;
import org.elasticsearch.cluster.service.MasterServiceTests;
@ -55,7 +56,6 @@ import org.junit.BeforeClass;
import java.util.ArrayList;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
import java.util.Optional;
import java.util.Random;
@ -189,9 +189,11 @@ public class NodeJoinTests extends ESTestCase {
}
protected DiscoveryNode newNode(int i, boolean master) {
Set<DiscoveryNode.Role> roles = new HashSet<>();
final Set<DiscoveryNodeRole> roles;
if (master) {
roles.add(DiscoveryNode.Role.MASTER);
roles = Collections.singleton(DiscoveryNodeRole.MASTER_ROLE);
} else {
roles = Collections.emptySet();
}
final String prefix = master ? "master_" : "data_";
return new DiscoveryNode(prefix + i, i + "", buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);

View File

@ -24,6 +24,7 @@ import org.elasticsearch.action.ActionListener;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.coordination.CoordinationMetaData.VotingConfiguration;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.common.Nullable;
import org.elasticsearch.common.collect.Tuple;
@ -464,13 +465,13 @@ public class PublicationTests extends ESTestCase {
attributes.put("custom", randomBoolean() ? "match" : randomAlphaOfLengthBetween(3, 5));
}
final DiscoveryNode node = newNode(i, attributes,
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))));
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)));
nodesList.add(node);
}
return nodesList;
}
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNode.Role> roles) {
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNodeRole> roles) {
return new DiscoveryNode("name_" + nodeId, "node_" + nodeId, buildNewFakeTransportAddress(), attributes, roles,
Version.CURRENT);
}

View File

@ -25,11 +25,11 @@ import org.elasticsearch.action.support.ActiveShardCount;
import org.elasticsearch.action.support.replication.ClusterStateCreationUtils;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
import org.elasticsearch.cluster.routing.ShardRoutingState;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.util.set.Sets;
import org.elasticsearch.indices.cluster.ClusterStateChanges;
import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.threadpool.TestThreadPool;
@ -104,11 +104,9 @@ public class AutoExpandReplicasTests extends ESTestCase {
private static final AtomicInteger nodeIdGenerator = new AtomicInteger();
protected DiscoveryNode createNode(DiscoveryNode.Role... mustHaveRoles) {
Set<DiscoveryNode.Role> roles = new HashSet<>(randomSubsetOf(Sets.newHashSet(DiscoveryNode.Role.values())));
for (DiscoveryNode.Role mustHaveRole : mustHaveRoles) {
roles.add(mustHaveRole);
}
protected DiscoveryNode createNode(DiscoveryNodeRole... mustHaveRoles) {
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
Collections.addAll(roles, mustHaveRoles);
final String id = String.format(Locale.ROOT, "node_%03d", nodeIdGenerator.incrementAndGet());
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(), roles,
Version.CURRENT);
@ -126,12 +124,12 @@ public class AutoExpandReplicasTests extends ESTestCase {
try {
List<DiscoveryNode> allNodes = new ArrayList<>();
DiscoveryNode localNode = createNode(DiscoveryNode.Role.MASTER); // local node is the master
DiscoveryNode localNode = createNode(DiscoveryNodeRole.MASTER_ROLE); // local node is the master
allNodes.add(localNode);
int numDataNodes = randomIntBetween(3, 5);
List<DiscoveryNode> dataNodes = new ArrayList<>(numDataNodes);
for (int i = 0; i < numDataNodes; i++) {
dataNodes.add(createNode(DiscoveryNode.Role.DATA));
dataNodes.add(createNode(DiscoveryNodeRole.DATA_ROLE));
}
allNodes.addAll(dataNodes);
ClusterState state = ClusterStateCreationUtils.state(localNode, localNode, allNodes.toArray(new DiscoveryNode[0]));
@ -179,7 +177,7 @@ public class AutoExpandReplicasTests extends ESTestCase {
.collect(Collectors.toList());
if (randomBoolean()) {
nodesToAdd.add(createNode(DiscoveryNode.Role.DATA));
nodesToAdd.add(createNode(DiscoveryNodeRole.DATA_ROLE));
}
state = cluster.joinNodesAndBecomeMaster(state, nodesToAdd);

View File

@ -27,6 +27,7 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.EmptyClusterInfoService;
import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.routing.ShardRoutingState;
@ -409,8 +410,9 @@ public class MetaDataCreateIndexServiceTests extends ESTestCase {
}
private DiscoveryNode newNode(String nodeId) {
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(),
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.DATA))), Version.CURRENT);
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE)));
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
}
public void testValidateIndexName() throws Exception {

View File

@ -30,6 +30,7 @@ import org.elasticsearch.cluster.SnapshotsInProgress;
import org.elasticsearch.cluster.block.ClusterBlock;
import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -152,9 +153,9 @@ public class MetaDataIndexStateServiceTests extends ESTestCase {
state = ClusterState.builder(state)
.nodes(DiscoveryNodes.builder(state.nodes())
.add(new DiscoveryNode("old_node", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.V_7_0_0))
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.V_7_0_0))
.add(new DiscoveryNode("new_node", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.V_7_2_0)))
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.V_7_2_0)))
.build();
state = MetaDataIndexStateService.closeRoutingTable(state, blockedIndices, results).v1();

View File

@ -30,6 +30,7 @@ import org.elasticsearch.client.IndicesAdminClient;
import org.elasticsearch.cluster.ClusterChangedEvent;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.bytes.BytesArray;
@ -46,7 +47,6 @@ import org.junit.Before;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
@ -247,8 +247,8 @@ public class TemplateUpgradeServiceTests extends ESTestCase {
assertThat(service.upgradesInProgress.get(), equalTo(2));
}
private static final Set<DiscoveryNode.Role> MASTER_DATA_ROLES =
Collections.unmodifiableSet(EnumSet.of(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.DATA));
private static final Set<DiscoveryNodeRole> MASTER_DATA_ROLES =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE)));
@SuppressWarnings("unchecked")
public void testClusterStateUpdate() throws InterruptedException {

View File

@ -0,0 +1,100 @@
/*
* Licensed to Elasticsearch under one or more contributor
* license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright
* ownership. Elasticsearch 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.elasticsearch.cluster.node;
import org.elasticsearch.action.admin.cluster.node.info.NodesInfoResponse;
import org.elasticsearch.common.settings.Setting;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.test.ESIntegTestCase;
import org.hamcrest.Matcher;
import java.util.Collection;
import java.util.Collections;
import java.util.List;
import java.util.Set;
import static org.hamcrest.Matchers.hasItem;
import static org.hamcrest.Matchers.hasSize;
import static org.hamcrest.Matchers.not;
public class DiscoveryNodeRoleIT extends ESIntegTestCase {
public static class AdditionalRolePlugin extends Plugin {
public AdditionalRolePlugin() {
}
static final Setting<Boolean> NODE_ADDITIONAL_SETTING =
Setting.boolSetting("node.additional", true, Setting.Property.NodeScope);
static DiscoveryNodeRole ADDITIONAL_ROLE = new DiscoveryNodeRole("additional", "a") {
@Override
protected Setting<Boolean> roleSetting() {
return NODE_ADDITIONAL_SETTING;
}
};
@Override
public Set<DiscoveryNodeRole> getRoles() {
return Collections.singleton(ADDITIONAL_ROLE);
}
@Override
public List<Setting<?>> getSettings() {
return Collections.singletonList(NODE_ADDITIONAL_SETTING);
}
}
@Override
protected Collection<Class<? extends Plugin>> nodePlugins() {
return Collections.singletonList(AdditionalRolePlugin.class);
}
public void testDefaultHasAdditionalRole() {
runTestNodeHasAdditionalRole(Settings.EMPTY);
}
public void testExplicitlyHasAdditionalRole() {
runTestNodeHasAdditionalRole(Settings.builder().put(AdditionalRolePlugin.NODE_ADDITIONAL_SETTING.getKey(), true).build());
}
public void testDoesNotHaveAdditionalRole() {
runTestNodeHasAdditionalRole(Settings.builder().put(AdditionalRolePlugin.NODE_ADDITIONAL_SETTING.getKey(), false).build());
}
private void runTestNodeHasAdditionalRole(final Settings settings) {
final String name = internalCluster().startNode(settings);
final NodesInfoResponse response = client().admin().cluster().prepareNodesInfo(name).get();
assertThat(response.getNodes(), hasSize(1));
final Matcher<Iterable<? super DiscoveryNodeRole>> matcher;
if (AdditionalRolePlugin.NODE_ADDITIONAL_SETTING.get(settings)) {
matcher = hasItem(AdditionalRolePlugin.ADDITIONAL_ROLE);
} else {
matcher = not(hasItem(AdditionalRolePlugin.ADDITIONAL_ROLE));
}
assertThat(response.getNodes().get(0).getNode().getRoles(), matcher);
}
}

View File

@ -0,0 +1,79 @@
/*
* Licensed to Elasticsearch under one or more contributor
* license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright
* ownership. Elasticsearch 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.elasticsearch.cluster.node;
import org.elasticsearch.common.settings.Setting;
import org.elasticsearch.test.ESTestCase;
import java.util.Arrays;
import java.util.HashSet;
import static org.hamcrest.Matchers.containsString;
import static org.hamcrest.Matchers.hasToString;
public class DiscoveryNodeRoleTests extends ESTestCase {
public void testDiscoveryNodeSetPossibleRolesRejectsDuplicateRoleNames() {
final IllegalStateException e = expectThrows(
IllegalStateException.class,
() -> DiscoveryNode.setPossibleRoles(new HashSet<>(Arrays.asList(
new DiscoveryNodeRole("foo", "f") {
@Override
protected Setting<Boolean> roleSetting() {
return null;
}
},
new DiscoveryNodeRole("foo", "f") {
@Override
protected Setting<Boolean> roleSetting() {
return null;
}
}))));
assertThat(e, hasToString(containsString("Duplicate key foo ")));
}
public void testDiscoveryNodeSetPossibleRolesRejectsDuplicateRoleNameAbbreviations() {
final IllegalStateException e = expectThrows(
IllegalStateException.class,
() -> DiscoveryNode.setPossibleRoles(new HashSet<>(Arrays.asList(
new DiscoveryNodeRole("foo_1", "f") {
@Override
protected Setting<Boolean> roleSetting() {
return null;
}
},
new DiscoveryNodeRole("foo_2", "f") {
@Override
protected Setting<Boolean> roleSetting() {
return null;
}
}))));
assertThat(e, hasToString(containsString("Duplicate key f ")));
}
}

View File

@ -241,7 +241,7 @@ public class DiscoveryNodesTests extends ESTestCase {
attributes.put("custom", randomBoolean() ? "match" : randomAlphaOfLengthBetween(3, 5));
}
final DiscoveryNode node = newNode(idGenerator.getAndIncrement(), attributes,
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))));
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)));
nodesList.add(node);
}
return nodesList;
@ -259,7 +259,7 @@ public class DiscoveryNodesTests extends ESTestCase {
return discoBuilder.build();
}
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNode.Role> roles) {
private static DiscoveryNode newNode(int nodeId, Map<String, String> attributes, Set<DiscoveryNodeRole> roles) {
return new DiscoveryNode("name_" + nodeId, "node_" + nodeId, buildNewFakeTransportAddress(), attributes, roles,
Version.CURRENT);
}
@ -275,21 +275,21 @@ public class DiscoveryNodesTests extends ESTestCase {
Set<String> matchingNodeIds(DiscoveryNodes nodes) {
return Collections.singleton(nodes.getMasterNodeId());
}
}, MASTER_ELIGIBLE(DiscoveryNode.Role.MASTER.getRoleName() + ":true") {
}, MASTER_ELIGIBLE(DiscoveryNodeRole.MASTER_ROLE.roleName() + ":true") {
@Override
Set<String> matchingNodeIds(DiscoveryNodes nodes) {
Set<String> ids = new HashSet<>();
nodes.getMasterNodes().keysIt().forEachRemaining(ids::add);
return ids;
}
}, DATA(DiscoveryNode.Role.DATA.getRoleName() + ":true") {
}, DATA(DiscoveryNodeRole.DATA_ROLE.roleName() + ":true") {
@Override
Set<String> matchingNodeIds(DiscoveryNodes nodes) {
Set<String> ids = new HashSet<>();
nodes.getDataNodes().keysIt().forEachRemaining(ids::add);
return ids;
}
}, INGEST(DiscoveryNode.Role.INGEST.getRoleName() + ":true") {
}, INGEST(DiscoveryNodeRole.INGEST_ROLE.roleName() + ":true") {
@Override
Set<String> matchingNodeIds(DiscoveryNodes nodes) {
Set<String> ids = new HashSet<>();
@ -328,13 +328,13 @@ public class DiscoveryNodesTests extends ESTestCase {
public void testMaxMinNodeVersion() {
DiscoveryNodes.Builder discoBuilder = DiscoveryNodes.builder();
discoBuilder.add(new DiscoveryNode("name_" + 1, "node_" + 1, buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))),
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)),
Version.fromString("5.1.0")));
discoBuilder.add(new DiscoveryNode("name_" + 2, "node_" + 2, buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))),
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)),
Version.fromString("6.3.0")));
discoBuilder.add(new DiscoveryNode("name_" + 3, "node_" + 3, buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))),
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)),
Version.fromString("1.1.0")));
discoBuilder.localNodeId("name_1");
discoBuilder.masterNodeId("name_2");

View File

@ -27,7 +27,7 @@ import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.ESAllocationTestCase;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.allocation.AllocationService;
import org.elasticsearch.cluster.service.ClusterService;
@ -223,7 +223,7 @@ public class DelayedAllocationServiceTests extends ESAllocationTestCase {
ClusterState clusterState = ClusterState.builder(ClusterName.CLUSTER_NAME_SETTING.getDefault(Settings.EMPTY)).metaData(metaData)
.routingTable(RoutingTable.builder().addAsNew(metaData.index("short_delay")).addAsNew(metaData.index("long_delay")).build())
.nodes(DiscoveryNodes.builder()
.add(newNode("node0", singleton(DiscoveryNode.Role.MASTER))).localNodeId("node0").masterNodeId("node0")
.add(newNode("node0", singleton(DiscoveryNodeRole.MASTER_ROLE))).localNodeId("node0").masterNodeId("node0")
.add(newNode("node1")).add(newNode("node2")).add(newNode("node3")).add(newNode("node4"))).build();
// allocate shards
clusterState = allocationService.reroute(clusterState, "reroute");

View File

@ -29,6 +29,7 @@ import org.elasticsearch.cluster.ESAllocationTestCase;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RecoverySource;
import org.elasticsearch.cluster.routing.RoutingNode;
@ -59,8 +60,8 @@ import org.elasticsearch.index.IndexNotFoundException;
import org.elasticsearch.index.shard.ShardId;
import org.elasticsearch.index.shard.ShardNotFoundException;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
@ -160,7 +161,7 @@ public class AllocationCommandsTests extends ESAllocationTestCase {
.add(newNode("node1"))
.add(newNode("node2"))
.add(newNode("node3"))
.add(newNode("node4", singleton(DiscoveryNode.Role.MASTER)))
.add(newNode("node4", singleton(DiscoveryNodeRole.MASTER_ROLE)))
).build();
clusterState = allocation.reroute(clusterState, "reroute");
assertThat(clusterState.getRoutingNodes().shardsWithState(INITIALIZING).size(), equalTo(0));
@ -620,7 +621,8 @@ public class AllocationCommandsTests extends ESAllocationTestCase {
DiscoveryNode node1 = new DiscoveryNode("node1", "node1", "node1", "test1", "test1", buildNewFakeTransportAddress(), emptyMap(),
MASTER_DATA_ROLES, Version.CURRENT);
DiscoveryNode node2 = new DiscoveryNode("node2", "node2", "node2", "test2", "test2", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(EnumSet.of(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.INGEST))), Version.CURRENT);
new HashSet<>(randomSubsetOf(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.INGEST_ROLE)))),
Version.CURRENT);
clusterState = ClusterState.builder(clusterState).nodes(
DiscoveryNodes.builder()
@ -659,7 +661,8 @@ public class AllocationCommandsTests extends ESAllocationTestCase {
DiscoveryNode node1 = new DiscoveryNode("node1", "node1", "node1", "test1", "test1", buildNewFakeTransportAddress(), emptyMap(),
MASTER_DATA_ROLES, Version.CURRENT);
DiscoveryNode node2 = new DiscoveryNode("node2", "node2", "node2", "test2", "test2", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(EnumSet.of(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.INGEST))), Version.CURRENT);
new HashSet<>(randomSubsetOf(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.INGEST_ROLE)))),
Version.CURRENT);
clusterState = ClusterState.builder(clusterState).nodes(
DiscoveryNodes.builder()

View File

@ -20,7 +20,6 @@
package org.elasticsearch.cluster.routing.allocation;
import com.carrotsearch.hppc.cursors.ObjectCursor;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
import org.elasticsearch.Version;
@ -33,6 +32,7 @@ import org.elasticsearch.cluster.ESAllocationTestCase;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RoutingNode;
import org.elasticsearch.cluster.routing.RoutingNodes;
@ -40,7 +40,6 @@ import org.elasticsearch.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.routing.ShardRouting;
import org.elasticsearch.cluster.routing.allocation.decider.ClusterRebalanceAllocationDecider;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.util.set.Sets;
import org.elasticsearch.indices.cluster.ClusterStateChanges;
import org.elasticsearch.test.VersionUtils;
import org.elasticsearch.threadpool.TestThreadPool;
@ -210,11 +209,11 @@ public class FailedNodeRoutingTests extends ESAllocationTestCase {
public ClusterState randomInitialClusterState() {
List<DiscoveryNode> allNodes = new ArrayList<>();
DiscoveryNode localNode = createNode(DiscoveryNode.Role.MASTER); // local node is the master
DiscoveryNode localNode = createNode(DiscoveryNodeRole.MASTER_ROLE); // local node is the master
allNodes.add(localNode);
// at least two nodes that have the data role so that we can allocate shards
allNodes.add(createNode(DiscoveryNode.Role.DATA));
allNodes.add(createNode(DiscoveryNode.Role.DATA));
allNodes.add(createNode(DiscoveryNodeRole.DATA_ROLE));
allNodes.add(createNode(DiscoveryNodeRole.DATA_ROLE));
for (int i = 0; i < randomIntBetween(2, 5); i++) {
allNodes.add(createNode());
}
@ -223,11 +222,9 @@ public class FailedNodeRoutingTests extends ESAllocationTestCase {
}
protected DiscoveryNode createNode(DiscoveryNode.Role... mustHaveRoles) {
Set<DiscoveryNode.Role> roles = new HashSet<>(randomSubsetOf(Sets.newHashSet(DiscoveryNode.Role.values())));
for (DiscoveryNode.Role mustHaveRole : mustHaveRoles) {
roles.add(mustHaveRole);
}
protected DiscoveryNode createNode(DiscoveryNodeRole... mustHaveRoles) {
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
Collections.addAll(roles, mustHaveRoles);
final String id = String.format(Locale.ROOT, "node_%03d", nodeIdGenerator.incrementAndGet());
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(), roles,
VersionUtils.randomVersionBetween(random(), Version.V_6_0_0_alpha1, null));

View File

@ -30,6 +30,7 @@ import org.elasticsearch.cluster.MockInternalClusterInfoService.DevNullClusterIn
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -912,9 +913,9 @@ public class DiskThresholdDeciderTests extends ESAllocationTestCase {
logger.info("--> adding one master node, one data node");
DiscoveryNode discoveryNode1 = new DiscoveryNode("", "node1", buildNewFakeTransportAddress(), emptyMap(),
singleton(DiscoveryNode.Role.MASTER), Version.CURRENT);
singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
DiscoveryNode discoveryNode2 = new DiscoveryNode("", "node2", buildNewFakeTransportAddress(), emptyMap(),
singleton(DiscoveryNode.Role.DATA), Version.CURRENT);
singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
DiscoveryNodes discoveryNodes = DiscoveryNodes.builder().add(discoveryNode1).add(discoveryNode2).build();
ClusterState baseClusterState = ClusterState.builder(ClusterName.CLUSTER_NAME_SETTING.getDefault(Settings.EMPTY))
@ -977,7 +978,7 @@ public class DiskThresholdDeciderTests extends ESAllocationTestCase {
// Add another datanode, it should relocate.
logger.info("--> adding node3");
DiscoveryNode discoveryNode3 = new DiscoveryNode("", "node3", buildNewFakeTransportAddress(), emptyMap(),
singleton(DiscoveryNode.Role.DATA), Version.CURRENT);
singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
ClusterState updateClusterState = ClusterState.builder(clusterState).nodes(DiscoveryNodes.builder(clusterState.nodes())
.add(discoveryNode3)).build();

View File

@ -29,6 +29,7 @@ import org.elasticsearch.cluster.MockInternalClusterInfoService.DevNullClusterIn
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RecoverySource.EmptyStoreRecoverySource;
import org.elasticsearch.cluster.routing.RecoverySource.LocalShardsRecoverySource;
@ -47,7 +48,6 @@ import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.index.Index;
import org.elasticsearch.index.shard.ShardId;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
@ -73,9 +73,9 @@ public class DiskThresholdDeciderUnitTests extends ESAllocationTestCase {
ShardRouting test_0 = ShardRouting.newUnassigned(new ShardId(index, 0), true, EmptyStoreRecoverySource.INSTANCE,
new UnassignedInfo(UnassignedInfo.Reason.INDEX_CREATED, "foo"));
DiscoveryNode node_0 = new DiscoveryNode("node_0", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
DiscoveryNode node_1 = new DiscoveryNode("node_1", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
RoutingTable routingTable = RoutingTable.builder()
.addAsNew(metaData.index("test"))
@ -129,9 +129,9 @@ public class DiskThresholdDeciderUnitTests extends ESAllocationTestCase {
ShardRouting test_0 = ShardRouting.newUnassigned(new ShardId(index, 0), true, EmptyStoreRecoverySource.INSTANCE,
new UnassignedInfo(UnassignedInfo.Reason.INDEX_CREATED, "foo"));
DiscoveryNode node_0 = new DiscoveryNode("node_0", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
DiscoveryNode node_1 = new DiscoveryNode("node_1", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
RoutingTable routingTable = RoutingTable.builder()
.addAsNew(metaData.index("test"))
@ -178,9 +178,9 @@ public class DiskThresholdDeciderUnitTests extends ESAllocationTestCase {
ImmutableOpenMap.Builder<ShardRouting, String> shardRoutingMap = ImmutableOpenMap.builder();
DiscoveryNode node_0 = new DiscoveryNode("node_0", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
DiscoveryNode node_1 = new DiscoveryNode("node_1", buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
MetaData metaData = MetaData.builder()
.put(IndexMetaData.builder("test").settings(settings(Version.CURRENT)).numberOfShards(1).numberOfReplicas(1))

View File

@ -27,6 +27,7 @@ import org.elasticsearch.cluster.RestoreInProgress;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -171,7 +172,7 @@ public class RestoreInProgressAllocationDeciderTests extends ESAllocationTestCas
.build();
DiscoveryNodes discoveryNodes = DiscoveryNodes.builder()
.add(newNode("master", Collections.singleton(DiscoveryNode.Role.MASTER)))
.add(newNode("master", Collections.singleton(DiscoveryNodeRole.MASTER_ROLE)))
.localNodeId("master")
.masterNodeId("master")
.build();

View File

@ -21,6 +21,7 @@ package org.elasticsearch.discovery.zen;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.discovery.zen.ElectMasterService.MasterCandidate;
import org.elasticsearch.test.ESTestCase;
@ -45,9 +46,9 @@ public class ElectMasterServiceTests extends ESTestCase {
int count = scaledRandomIntBetween(1, 100);
ArrayList<DiscoveryNode> nodes = new ArrayList<>(count);
for (int i = 0; i < count; i++) {
Set<DiscoveryNode.Role> roles = new HashSet<>();
Set<DiscoveryNodeRole> roles = new HashSet<>();
if (randomBoolean()) {
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
}
DiscoveryNode node = new DiscoveryNode("n_" + i, "n_" + i, buildNewFakeTransportAddress(), Collections.emptyMap(),
roles, Version.CURRENT);
@ -62,8 +63,8 @@ public class ElectMasterServiceTests extends ESTestCase {
int count = scaledRandomIntBetween(1, 100);
ArrayList<MasterCandidate> candidates = new ArrayList<>(count);
for (int i = 0; i < count; i++) {
Set<DiscoveryNode.Role> roles = new HashSet<>();
roles.add(DiscoveryNode.Role.MASTER);
Set<DiscoveryNodeRole> roles = new HashSet<>();
roles.add(DiscoveryNodeRole.MASTER_ROLE);
DiscoveryNode node = new DiscoveryNode("n_" + i, "n_" + i, buildNewFakeTransportAddress(), Collections.emptyMap(),
roles, Version.CURRENT);
candidates.add(

View File

@ -29,6 +29,7 @@ import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -55,9 +56,7 @@ import org.junit.Before;
import org.junit.BeforeClass;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
@ -126,7 +125,7 @@ public class NodeJoinControllerTests extends ESTestCase {
private static ClusterState initialState(boolean withMaster) {
DiscoveryNode localNode = new DiscoveryNode("node", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())),Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES,Version.CURRENT);
ClusterState initialClusterState = ClusterState.builder(new ClusterName(ClusterServiceUtils.class.getSimpleName()))
.nodes(DiscoveryNodes.builder()
.add(localNode)
@ -570,7 +569,7 @@ public class NodeJoinControllerTests extends ESTestCase {
existing.getId(),
randomBoolean() ? existing.getAddress() : buildNewFakeTransportAddress(),
randomBoolean() ? existing.getAttributes() : Collections.singletonMap("attr", "other"),
randomBoolean() ? existing.getRoles() : new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))),
randomBoolean() ? existing.getRoles() : new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)),
existing.getVersion());
ExecutionException e = expectThrows(ExecutionException.class, () -> joinNode(other_node));
@ -599,12 +598,12 @@ public class NodeJoinControllerTests extends ESTestCase {
badVersion = randomFrom(allVersions().stream().filter(v -> v.major < Version.CURRENT.major).collect(Collectors.toList()));
}
final DiscoveryNode badNode = new DiscoveryNode("badNode", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))), badVersion);
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)), badVersion);
final Version goodVersion =
randomFrom(allVersions().stream().filter(v -> v.major >= Version.CURRENT.major).collect(Collectors.toList()));
final DiscoveryNode goodNode = new DiscoveryNode("goodNode", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))), goodVersion);
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)), goodVersion);
CountDownLatch latch = new CountDownLatch(1);
// block cluster state
@ -647,11 +646,11 @@ public class NodeJoinControllerTests extends ESTestCase {
setState(masterService, builder.build());
final Version badVersion = getPreviousVersion(Version.CURRENT.minimumCompatibilityVersion());
final DiscoveryNode badNode = new DiscoveryNode("badNode", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))), badVersion);
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)), badVersion);
final Version goodVersion = randomFrom(randomCompatibleVersion(random(), Version.CURRENT));
final DiscoveryNode goodNode = new DiscoveryNode("goodNode", buildNewFakeTransportAddress(), emptyMap(),
new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))), goodVersion);
new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)), goodVersion);
CountDownLatch latch = new CountDownLatch(1);
// block cluster state
@ -694,7 +693,7 @@ public class NodeJoinControllerTests extends ESTestCase {
ClusterState initialState = initialState(true);
final DiscoveryNode masterNode = initialState.nodes().getLocalNode();
final DiscoveryNode otherNode = new DiscoveryNode("other_node", buildNewFakeTransportAddress(), emptyMap(),
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
// simulate master going down with stale nodes in it's cluster state (for example when min master nodes is set to 2)
// also add some shards to that node
DiscoveryNodes.Builder discoBuilder = DiscoveryNodes.builder(initialState.nodes());
@ -781,7 +780,7 @@ public class NodeJoinControllerTests extends ESTestCase {
final DiscoveryNodes.Builder nodesBuilder = DiscoveryNodes.builder(state.nodes());
for (int i = 0;i< count;i++) {
final DiscoveryNode node = new DiscoveryNode("node_" + state.nodes().getSize() + i, buildNewFakeTransportAddress(),
emptyMap(), new HashSet<>(randomSubsetOf(Arrays.asList(DiscoveryNode.Role.values()))), Version.CURRENT);
emptyMap(), new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES)), Version.CURRENT);
nodesBuilder.add(node);
}
setupMasterServiceAndNodeJoinController(ClusterState.builder(state).nodes(nodesBuilder).build());
@ -858,9 +857,11 @@ public class NodeJoinControllerTests extends ESTestCase {
}
protected DiscoveryNode newNode(int i, boolean master) {
Set<DiscoveryNode.Role> roles = new HashSet<>();
final Set<DiscoveryNodeRole> roles;
if (master) {
roles.add(DiscoveryNode.Role.MASTER);
roles = Collections.singleton(DiscoveryNodeRole.MASTER_ROLE);
} else {
roles = Collections.emptySet();
}
final String prefix = master ? "master_" : "data_";
return new DiscoveryNode(prefix + i, i + "", buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);

View File

@ -28,13 +28,14 @@ import org.elasticsearch.cluster.ClusterName;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.ClusterStateTaskExecutor;
import org.elasticsearch.cluster.ESAllocationTestCase;
import org.elasticsearch.cluster.coordination.FailedToCommitClusterStateException;
import org.elasticsearch.cluster.coordination.JoinTaskExecutor;
import org.elasticsearch.cluster.coordination.NodeRemovalClusterStateTaskExecutor;
import org.elasticsearch.cluster.coordination.ValidateJoinRequest;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -49,7 +50,6 @@ import org.elasticsearch.common.io.stream.NamedWriteableRegistry;
import org.elasticsearch.common.settings.ClusterSettings;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.core.internal.io.IOUtils;
import org.elasticsearch.cluster.coordination.FailedToCommitClusterStateException;
import org.elasticsearch.discovery.zen.PublishClusterStateActionTests.AssertingAckListener;
import org.elasticsearch.discovery.zen.ZenDiscovery.ZenNodeRemovalClusterStateTaskExecutor;
import org.elasticsearch.gateway.GatewayMetaState;
@ -72,7 +72,6 @@ import java.util.ArrayDeque;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Locale;
@ -174,7 +173,7 @@ public class ZenDiscoveryUnitTests extends ESTestCase {
ArrayList<DiscoveryNode> masterNodes = new ArrayList<>();
ArrayList<DiscoveryNode> allNodes = new ArrayList<>();
for (int i = randomIntBetween(10, 20); i >= 0; i--) {
Set<Role> roles = new HashSet<>(randomSubsetOf(Arrays.asList(Role.values())));
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
DiscoveryNode node = new DiscoveryNode("node_" + i, "id_" + i, buildNewFakeTransportAddress(), Collections.emptyMap(),
roles, Version.CURRENT);
responses.add(new ZenPing.PingResponse(node, randomBoolean() ? null : node, new ClusterName("test"), randomLong()));
@ -392,9 +391,9 @@ public class ZenDiscoveryUnitTests extends ESTestCase {
for (int i = 0; i < iters; i++) {
ClusterState.Builder stateBuilder = ClusterState.builder(ClusterName.DEFAULT);
final DiscoveryNode otherNode = new DiscoveryNode("other_node", buildNewFakeTransportAddress(), emptyMap(),
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
final DiscoveryNode localNode = new DiscoveryNode("other_node", buildNewFakeTransportAddress(), emptyMap(),
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
MembershipAction.ValidateJoinRequestRequestHandler request = new MembershipAction.ValidateJoinRequestRequestHandler
(() -> localNode, JoinTaskExecutor.addBuiltInJoinValidators(Collections.emptyList()));
final boolean incompatible = randomBoolean();

View File

@ -23,6 +23,7 @@ import org.elasticsearch.Version;
import org.elasticsearch.action.FailedNodeException;
import org.elasticsearch.action.support.nodes.BaseNodeResponse;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.index.shard.ShardId;
import org.elasticsearch.test.ESTestCase;
@ -43,11 +44,11 @@ import static org.hamcrest.Matchers.sameInstance;
public class AsyncShardFetchTests extends ESTestCase {
private final DiscoveryNode node1 = new DiscoveryNode("node1", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.DATA), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
private final Response response1 = new Response(node1);
private final Throwable failure1 = new Throwable("simulated failure 1");
private final DiscoveryNode node2 = new DiscoveryNode("node2", buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.DATA), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
private final Response response2 = new Response(node2);
private final Throwable failure2 = new Throwable("simulate failure 2");

View File

@ -26,6 +26,7 @@ import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.metadata.MetaDataIndexStateService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.UUIDs;
import org.elasticsearch.common.settings.ClusterSettings;
@ -279,7 +280,7 @@ public class ClusterStateUpdatersTests extends ESTestCase {
.metaData(metaData)
.build();
final DiscoveryNode localNode = new DiscoveryNode("node1", buildNewFakeTransportAddress(), Collections.emptyMap(),
Sets.newHashSet(DiscoveryNode.Role.MASTER), Version.CURRENT);
Sets.newHashSet(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
final ClusterState updatedState = setLocalNode(initialState, localNode);
@ -322,7 +323,7 @@ public class ClusterStateUpdatersTests extends ESTestCase {
.blocks(ClusterBlocks.builder().addGlobalBlock(STATE_NOT_RECOVERED_BLOCK))
.build();
final DiscoveryNode localNode = new DiscoveryNode("node1", buildNewFakeTransportAddress(), Collections.emptyMap(),
Sets.newHashSet(DiscoveryNode.Role.MASTER), Version.CURRENT);
Sets.newHashSet(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
final ClusterState updatedState = Function.<ClusterState>identity()
.andThen(state -> setLocalNode(state, localNode))
.andThen(ClusterStateUpdaters::recoverClusterBlocks)

View File

@ -28,6 +28,7 @@ import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.Manifest;
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.settings.Settings;
import org.elasticsearch.common.util.set.Sets;
@ -50,7 +51,7 @@ public class GatewayMetaStatePersistedStateTests extends ESTestCase {
public void setUp() throws Exception {
nodeEnvironment = newNodeEnvironment();
localNode = new DiscoveryNode("node1", buildNewFakeTransportAddress(), Collections.emptyMap(),
Sets.newHashSet(DiscoveryNode.Role.MASTER), Version.CURRENT);
Sets.newHashSet(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
clusterName = new ClusterName(randomAlphaOfLength(10));
settings = Settings.builder().put(ClusterName.CLUSTER_NAME_SETTING.getKey(), clusterName.value()).build();
super.setUp();

View File

@ -29,7 +29,7 @@ import org.elasticsearch.cluster.metadata.IndexTemplateMetaData;
import org.elasticsearch.cluster.metadata.Manifest;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.metadata.MetaDataIndexUpgradeService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.routing.allocation.AllocationService;
@ -144,7 +144,7 @@ public class GatewayMetaStateTests extends ESAllocationTestCase {
}
private DiscoveryNodes.Builder generateDiscoveryNodes(boolean masterEligible) {
Set<DiscoveryNode.Role> dataOnlyRoles = Collections.singleton(DiscoveryNode.Role.DATA);
Set<DiscoveryNodeRole> dataOnlyRoles = Collections.singleton(DiscoveryNodeRole.DATA_ROLE);
return DiscoveryNodes.builder().add(newNode("node1", masterEligible ? MASTER_DATA_ROLES : dataOnlyRoles))
.add(newNode("master_node", MASTER_DATA_ROLES)).localNodeId("node1").masterNodeId(masterEligible ? "node1" : "master_node");
}

View File

@ -26,10 +26,10 @@ import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.MetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.test.ESTestCase;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
@ -44,7 +44,7 @@ public class FakeThreadPoolMasterServiceTests extends ESTestCase {
List<Runnable> runnableTasks = new ArrayList<>();
AtomicReference<ClusterState> lastClusterStateRef = new AtomicReference<>();
DiscoveryNode discoveryNode = new DiscoveryNode("node", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Arrays.asList(DiscoveryNode.Role.values())), Version.CURRENT);
new HashSet<>(DiscoveryNodeRole.BUILT_IN_ROLES), Version.CURRENT);
lastClusterStateRef.set(ClusterStateCreationUtils.state(discoveryNode, discoveryNode));
long firstClusterStateVersion = lastClusterStateRef.get().version();
AtomicReference<ActionListener<Void>> publishingCallback = new AtomicReference<>();

View File

@ -38,6 +38,7 @@ import org.elasticsearch.cluster.coordination.NoMasterBlockService;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.routing.ShardRouting;
@ -257,11 +258,11 @@ public class IndicesClusterStateServiceRandomUpdatesTests extends AbstractIndice
public ClusterState randomInitialClusterState(Map<DiscoveryNode, IndicesClusterStateService> clusterStateServiceMap,
Supplier<MockIndicesService> indicesServiceSupplier) {
List<DiscoveryNode> allNodes = new ArrayList<>();
DiscoveryNode localNode = createNode(DiscoveryNode.Role.MASTER); // local node is the master
DiscoveryNode localNode = createNode(DiscoveryNodeRole.MASTER_ROLE); // local node is the master
allNodes.add(localNode);
// at least two nodes that have the data role so that we can allocate shards
allNodes.add(createNode(DiscoveryNode.Role.DATA));
allNodes.add(createNode(DiscoveryNode.Role.DATA));
allNodes.add(createNode(DiscoveryNodeRole.DATA_ROLE));
allNodes.add(createNode(DiscoveryNodeRole.DATA_ROLE));
for (int i = 0; i < randomIntBetween(2, 5); i++) {
allNodes.add(createNode());
}
@ -436,11 +437,9 @@ public class IndicesClusterStateServiceRandomUpdatesTests extends AbstractIndice
private static final AtomicInteger nodeIdGenerator = new AtomicInteger();
protected DiscoveryNode createNode(DiscoveryNode.Role... mustHaveRoles) {
Set<DiscoveryNode.Role> roles = new HashSet<>(randomSubsetOf(Sets.newHashSet(DiscoveryNode.Role.values())));
for (DiscoveryNode.Role mustHaveRole : mustHaveRoles) {
roles.add(mustHaveRole);
}
protected DiscoveryNode createNode(DiscoveryNodeRole... mustHaveRoles) {
Set<DiscoveryNodeRole> roles = new HashSet<>(randomSubsetOf(DiscoveryNodeRole.BUILT_IN_ROLES));
Collections.addAll(roles, mustHaveRoles);
final String id = String.format(Locale.ROOT, "node_%03d", nodeIdGenerator.incrementAndGet());
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(), roles,
Version.CURRENT);

View File

@ -30,6 +30,7 @@ import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.RoutingTable;
import org.elasticsearch.cluster.service.ClusterService;
@ -55,6 +56,7 @@ import java.util.Arrays;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicReference;
import java.util.function.BiFunction;
@ -791,9 +793,9 @@ public class PersistentTasksClusterServiceTests extends ESTestCase {
}
private DiscoveryNode newNode(String nodeId) {
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(),
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.DATA))),
Version.CURRENT);
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE)));
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
}
private ClusterState initialState() {

View File

@ -97,6 +97,7 @@ import org.elasticsearch.cluster.metadata.MetaDataDeleteIndexService;
import org.elasticsearch.cluster.metadata.MetaDataIndexUpgradeService;
import org.elasticsearch.cluster.metadata.MetaDataMappingService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.RoutingService;
import org.elasticsearch.cluster.routing.ShardRouting;
@ -804,14 +805,14 @@ public class SnapshotResiliencyTests extends ESTestCase {
}
private TestClusterNode newMasterNode(String nodeName) throws IOException {
return newNode(nodeName, DiscoveryNode.Role.MASTER);
return newNode(nodeName, DiscoveryNodeRole.MASTER_ROLE);
}
private TestClusterNode newDataNode(String nodeName) throws IOException {
return newNode(nodeName, DiscoveryNode.Role.DATA);
return newNode(nodeName, DiscoveryNodeRole.DATA_ROLE);
}
private TestClusterNode newNode(String nodeName, DiscoveryNode.Role role) throws IOException {
private TestClusterNode newNode(String nodeName, DiscoveryNodeRole role) throws IOException {
return new TestClusterNode(
new DiscoveryNode(nodeName, randomAlphaOfLength(10), buildNewFakeTransportAddress(), emptyMap(),
Collections.singleton(role), Version.CURRENT), this::getDisruption);

View File

@ -23,6 +23,7 @@ import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.OriginalIndices;
import org.elasticsearch.action.support.IndicesOptions;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.Strings;
import org.elasticsearch.common.collect.Tuple;
import org.elasticsearch.common.settings.AbstractScopedSettings;
@ -42,7 +43,6 @@ import java.net.InetAddress;
import java.net.InetSocketAddress;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
@ -776,49 +776,54 @@ public class RemoteClusterServiceTests extends ESTestCase {
Predicate<DiscoveryNode> nodePredicate = RemoteClusterService.getNodePredicate(Settings.EMPTY);
{
DiscoveryNode all = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.allOf(DiscoveryNode.Role.class)), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
assertTrue(nodePredicate.test(all));
}
{
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.DATA_ROLE, DiscoveryNodeRole.MASTER_ROLE)));
DiscoveryNode dataMaster = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.DATA, DiscoveryNode.Role.MASTER)), Version.CURRENT);
roles, Version.CURRENT);
assertTrue(nodePredicate.test(dataMaster));
}
{
DiscoveryNode dedicatedMaster = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.MASTER)), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.MASTER_ROLE), Version.CURRENT);
assertFalse(nodePredicate.test(dedicatedMaster));
}
{
DiscoveryNode dedicatedIngest = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.INGEST)), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.INGEST_ROLE), Version.CURRENT);
assertTrue(nodePredicate.test(dedicatedIngest));
}
{
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.INGEST_ROLE, DiscoveryNodeRole.MASTER_ROLE)));
DiscoveryNode masterIngest = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.INGEST, DiscoveryNode.Role.MASTER)), Version.CURRENT);
roles, Version.CURRENT);
assertTrue(nodePredicate.test(masterIngest));
}
{
DiscoveryNode dedicatedData = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.DATA)), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
assertTrue(nodePredicate.test(dedicatedData));
}
{
final Set<DiscoveryNodeRole> roles =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.DATA_ROLE, DiscoveryNodeRole.INGEST_ROLE)));
DiscoveryNode ingestData = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.of(DiscoveryNode.Role.DATA, DiscoveryNode.Role.INGEST)), Version.CURRENT);
roles, Version.CURRENT);
assertTrue(nodePredicate.test(ingestData));
}
{
DiscoveryNode coordOnly = new DiscoveryNode("id", address, Collections.emptyMap(),
new HashSet<>(EnumSet.noneOf(DiscoveryNode.Role.class)), Version.CURRENT);
DiscoveryNode coordOnly = new DiscoveryNode("id", address, Collections.emptyMap(), Collections.emptySet(), Version.CURRENT);
assertTrue(nodePredicate.test(coordOnly));
}
}
public void testGetNodePredicateNodeVersion() {
TransportAddress address = new TransportAddress(TransportAddress.META_ADDRESS, 0);
Set<DiscoveryNode.Role> roles = new HashSet<>(EnumSet.allOf(DiscoveryNode.Role.class));
Set<DiscoveryNodeRole> roles = DiscoveryNodeRole.BUILT_IN_ROLES;
Predicate<DiscoveryNode> nodePredicate = RemoteClusterService.getNodePredicate(Settings.EMPTY);
Version version = VersionUtils.randomVersion(random());
DiscoveryNode node = new DiscoveryNode("id", address, Collections.emptyMap(), roles, version);
@ -827,7 +832,7 @@ public class RemoteClusterServiceTests extends ESTestCase {
public void testGetNodePredicateNodeAttrs() {
TransportAddress address = new TransportAddress(TransportAddress.META_ADDRESS, 0);
Set<DiscoveryNode.Role> roles = new HashSet<>(EnumSet.allOf(DiscoveryNode.Role.class));
Set<DiscoveryNodeRole> roles = DiscoveryNodeRole.BUILT_IN_ROLES;
Settings settings = Settings.builder().put("cluster.remote.node.attr", "gateway").build();
Predicate<DiscoveryNode> nodePredicate = RemoteClusterService.getNodePredicate(settings);
{
@ -853,8 +858,8 @@ public class RemoteClusterServiceTests extends ESTestCase {
TransportAddress address = new TransportAddress(TransportAddress.META_ADDRESS, 0);
Settings settings = Settings.builder().put("cluster.remote.node.attr", "gateway").build();
Predicate<DiscoveryNode> nodePredicate = RemoteClusterService.getNodePredicate(settings);
Set<DiscoveryNode.Role> allRoles = new HashSet<>(EnumSet.allOf(DiscoveryNode.Role.class));
Set<DiscoveryNode.Role> dedicatedMasterRoles = new HashSet<>(EnumSet.of(DiscoveryNode.Role.MASTER));
Set<DiscoveryNodeRole> allRoles = DiscoveryNodeRole.BUILT_IN_ROLES;
Set<DiscoveryNodeRole> dedicatedMasterRoles = Collections.singleton(DiscoveryNodeRole.MASTER_ROLE);
{
DiscoveryNode node = new DiscoveryNode("id", address, Collections.singletonMap("gateway", "true"),
dedicatedMasterRoles, Version.CURRENT);

View File

@ -21,6 +21,7 @@ package org.elasticsearch.cluster;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.RoutingNode;
import org.elasticsearch.cluster.routing.RoutingNodes;
import org.elasticsearch.cluster.routing.ShardRouting;
@ -43,7 +44,7 @@ import org.elasticsearch.test.gateway.TestGatewayAllocator;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Random;
@ -94,8 +95,8 @@ public abstract class ESAllocationTestCase extends ESTestCase {
return new AllocationDeciders(deciders);
}
protected static Set<DiscoveryNode.Role> MASTER_DATA_ROLES =
Collections.unmodifiableSet(EnumSet.of(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.DATA));
protected static Set<DiscoveryNodeRole> MASTER_DATA_ROLES =
Collections.unmodifiableSet(new HashSet<>(Arrays.asList(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE)));
protected static DiscoveryNode newNode(String nodeId) {
return newNode(nodeId, Version.CURRENT);
@ -109,7 +110,7 @@ public abstract class ESAllocationTestCase extends ESTestCase {
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), attributes, MASTER_DATA_ROLES, Version.CURRENT);
}
protected static DiscoveryNode newNode(String nodeId, Set<DiscoveryNode.Role> roles) {
protected static DiscoveryNode newNode(String nodeId, Set<DiscoveryNodeRole> roles) {
return new DiscoveryNode(nodeId, buildNewFakeTransportAddress(), emptyMap(), roles, Version.CURRENT);
}

View File

@ -49,6 +49,7 @@ import org.elasticsearch.action.support.replication.TransportWriteAction;
import org.elasticsearch.action.support.replication.TransportWriteActionTestHelper;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.AllocationId;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
import org.elasticsearch.cluster.routing.RecoverySource;
@ -158,7 +159,7 @@ public abstract class ESIndexLevelReplicationTestCase extends IndexShardTestCase
protected DiscoveryNode getDiscoveryNode(String id) {
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(),
Collections.singleton(DiscoveryNode.Role.DATA), Version.CURRENT);
Collections.singleton(DiscoveryNodeRole.DATA_ROLE), Version.CURRENT);
}
protected class ReplicationGroup implements AutoCloseable, Iterable<IndexShard> {

View File

@ -27,6 +27,7 @@ import org.elasticsearch.action.support.PlainActionFuture;
import org.elasticsearch.action.support.replication.TransportReplicationAction;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
import org.elasticsearch.cluster.routing.RecoverySource;
import org.elasticsearch.cluster.routing.ShardRouting;
@ -86,7 +87,6 @@ import org.elasticsearch.threadpool.ThreadPool;
import java.io.IOException;
import java.util.Arrays;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
@ -562,7 +562,7 @@ public abstract class IndexShardTestCase extends ESTestCase {
}
protected DiscoveryNode getFakeDiscoNode(String id) {
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(), EnumSet.allOf(DiscoveryNode.Role.class),
return new DiscoveryNode(id, id, buildNewFakeTransportAddress(), Collections.emptyMap(), DiscoveryNodeRole.BUILT_IN_ROLES,
Version.CURRENT);
}

View File

@ -26,7 +26,9 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.NodeConnectionsService;
import org.elasticsearch.cluster.block.ClusterBlocks;
import org.elasticsearch.cluster.coordination.ClusterStatePublisher;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterApplier;
import org.elasticsearch.cluster.service.ClusterApplier.ClusterApplyListener;
@ -35,11 +37,9 @@ import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.cluster.service.MasterService;
import org.elasticsearch.common.settings.ClusterSettings;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.cluster.coordination.ClusterStatePublisher;
import org.elasticsearch.threadpool.ThreadPool;
import java.util.Collections;
import java.util.EnumSet;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.atomic.AtomicReference;
@ -123,7 +123,7 @@ public class ClusterServiceUtils {
public static ClusterService createClusterService(ThreadPool threadPool) {
DiscoveryNode discoveryNode = new DiscoveryNode("node", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
return createClusterService(threadPool, discoveryNode);
}

View File

@ -52,6 +52,8 @@ import org.elasticsearch.bootstrap.JavaVersion;
import org.elasticsearch.client.Requests;
import org.elasticsearch.cluster.ClusterModule;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.CheckedBiFunction;
import org.elasticsearch.common.CheckedRunnable;
import org.elasticsearch.common.SuppressForbidden;
@ -348,6 +350,16 @@ public abstract class ESTestCase extends LuceneTestCase {
}
}
@BeforeClass
public static void setPossibleRoles() {
DiscoveryNode.setPossibleRoles(DiscoveryNodeRole.BUILT_IN_ROLES);
}
@AfterClass
public static void clearPossibleRoles() {
DiscoveryNode.setPossibleRoles(Collections.emptySet());
}
/**
* Whether or not we check after each test whether it has left warnings behind. That happens if any deprecated feature or syntax
* was used by the test and the test didn't assert on it using {@link #assertWarnings(String...)}.

View File

@ -46,7 +46,7 @@ import org.elasticsearch.cluster.action.index.MappingUpdatedAction;
import org.elasticsearch.cluster.coordination.ClusterBootstrapService;
import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNode.Role;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -154,8 +154,8 @@ import static org.elasticsearch.discovery.DiscoveryModule.DISCOVERY_TYPE_SETTING
import static org.elasticsearch.discovery.DiscoveryModule.ZEN2_DISCOVERY_TYPE;
import static org.elasticsearch.discovery.DiscoveryModule.ZEN_DISCOVERY_TYPE;
import static org.elasticsearch.discovery.DiscoverySettings.INITIAL_STATE_TIMEOUT_SETTING;
import static org.elasticsearch.discovery.zen.ElectMasterService.DISCOVERY_ZEN_MINIMUM_MASTER_NODES_SETTING;
import static org.elasticsearch.discovery.FileBasedSeedHostsProvider.UNICAST_HOSTS_FILE;
import static org.elasticsearch.discovery.zen.ElectMasterService.DISCOVERY_ZEN_MINIMUM_MASTER_NODES_SETTING;
import static org.elasticsearch.test.ESTestCase.assertBusy;
import static org.elasticsearch.test.ESTestCase.awaitBusy;
import static org.elasticsearch.test.ESTestCase.getTestTransportType;
@ -736,10 +736,10 @@ public final class InternalTestCluster extends TestCluster {
private static String getRoleSuffix(Settings settings) {
String suffix = "";
if (Node.NODE_MASTER_SETTING.exists(settings) && Node.NODE_MASTER_SETTING.get(settings)) {
suffix = suffix + Role.MASTER.getAbbreviation();
suffix = suffix + DiscoveryNodeRole.MASTER_ROLE.roleNameAbbreviation();
}
if (Node.NODE_DATA_SETTING.exists(settings) && Node.NODE_DATA_SETTING.get(settings)) {
suffix = suffix + Role.DATA.getAbbreviation();
suffix = suffix + DiscoveryNodeRole.DATA_ROLE.roleNameAbbreviation();
}
if (Node.NODE_MASTER_SETTING.exists(settings) && Node.NODE_MASTER_SETTING.get(settings) == false &&
Node.NODE_DATA_SETTING.exists(settings) && Node.NODE_DATA_SETTING.get(settings) == false
@ -1891,7 +1891,7 @@ public final class InternalTestCluster extends TestCluster {
public synchronized void fullRestart(RestartCallback callback) throws Exception {
int numNodesRestarted = 0;
final Settings[] newNodeSettings = new Settings[nextNodeId.get()];
Map<Set<Role>, List<NodeAndClient>> nodesByRoles = new HashMap<>();
Map<Set<DiscoveryNodeRole>, List<NodeAndClient>> nodesByRoles = new HashMap<>();
Set[] rolesOrderedByOriginalStartupOrder = new Set[nextNodeId.get()];
final int minMasterNodes = autoManageMasterNodes ? getMinMasterNodes(getMasterNodesCount()) : -1;
for (NodeAndClient nodeAndClient : nodes.values()) {

View File

@ -21,6 +21,7 @@ package org.elasticsearch.test.test;
import org.apache.lucene.util.LuceneTestCase;
import org.elasticsearch.client.Client;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.network.NetworkModule;
import org.elasticsearch.common.settings.Settings;
@ -53,9 +54,6 @@ import java.util.Set;
import java.util.function.Function;
import java.util.stream.Collectors;
import static org.elasticsearch.cluster.node.DiscoveryNode.Role.DATA;
import static org.elasticsearch.cluster.node.DiscoveryNode.Role.INGEST;
import static org.elasticsearch.cluster.node.DiscoveryNode.Role.MASTER;
import static org.elasticsearch.discovery.DiscoveryModule.DISCOVERY_SEED_PROVIDERS_SETTING;
import static org.elasticsearch.test.hamcrest.ElasticsearchAssertions.assertFileExists;
import static org.elasticsearch.test.hamcrest.ElasticsearchAssertions.assertFileNotExists;
@ -337,33 +335,30 @@ public class InternalTestClusterTests extends ESTestCase {
}
}, 0, "", mockPlugins(), Function.identity());
cluster.beforeTest(random(), 0.0);
List<DiscoveryNode.Role> roles = new ArrayList<>();
List<DiscoveryNodeRole> roles = new ArrayList<>();
for (int i = 0; i < numNodes; i++) {
final DiscoveryNode.Role role = i == numNodes - 1 && roles.contains(MASTER) == false ?
MASTER : // last node and still no master
randomFrom(MASTER, DiscoveryNode.Role.DATA, DiscoveryNode.Role.INGEST);
final DiscoveryNodeRole role = i == numNodes - 1 && roles.contains(DiscoveryNodeRole.MASTER_ROLE) == false ?
DiscoveryNodeRole.MASTER_ROLE : // last node and still no master
randomFrom(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.DATA_ROLE, DiscoveryNodeRole.INGEST_ROLE);
roles.add(role);
}
cluster.setBootstrapMasterNodeIndex(randomIntBetween(0, (int) roles.stream().filter(role -> role.equals(MASTER)).count() - 1));
cluster.setBootstrapMasterNodeIndex(
randomIntBetween(0, (int) roles.stream().filter(role -> role.equals(DiscoveryNodeRole.MASTER_ROLE)).count() - 1));
try {
Map<DiscoveryNode.Role, Set<String>> pathsPerRole = new HashMap<>();
Map<DiscoveryNodeRole, Set<String>> pathsPerRole = new HashMap<>();
for (int i = 0; i < numNodes; i++) {
final DiscoveryNode.Role role = roles.get(i);
final DiscoveryNodeRole role = roles.get(i);
final String node;
switch (role) {
case MASTER:
node = cluster.startMasterOnlyNode();
break;
case DATA:
node = cluster.startDataOnlyNode();
break;
case INGEST:
node = cluster.startCoordinatingOnlyNode(Settings.EMPTY);
break;
default:
throw new IllegalStateException("get your story straight");
if (role == DiscoveryNodeRole.MASTER_ROLE) {
node = cluster.startMasterOnlyNode();
} else if (role == DiscoveryNodeRole.DATA_ROLE) {
node = cluster.startDataOnlyNode();
} else if (role == DiscoveryNodeRole.INGEST_ROLE) {
node = cluster.startCoordinatingOnlyNode(Settings.EMPTY);
} else {
throw new IllegalStateException("get your story straight");
}
Set<String> rolePaths = pathsPerRole.computeIfAbsent(role, k -> new HashSet<>());
for (Path path : getNodePaths(cluster, node)) {
@ -373,21 +368,21 @@ public class InternalTestClusterTests extends ESTestCase {
cluster.validateClusterFormed();
cluster.fullRestart();
Map<DiscoveryNode.Role, Set<String>> result = new HashMap<>();
Map<DiscoveryNodeRole, Set<String>> result = new HashMap<>();
for (String name : cluster.getNodeNames()) {
DiscoveryNode node = cluster.getInstance(ClusterService.class, name).localNode();
List<String> paths = Arrays.stream(getNodePaths(cluster, name)).map(Path::toString).collect(Collectors.toList());
if (node.isMasterNode()) {
result.computeIfAbsent(MASTER, k -> new HashSet<>()).addAll(paths);
result.computeIfAbsent(DiscoveryNodeRole.MASTER_ROLE, k -> new HashSet<>()).addAll(paths);
} else if (node.isDataNode()) {
result.computeIfAbsent(DATA, k -> new HashSet<>()).addAll(paths);
result.computeIfAbsent(DiscoveryNodeRole.DATA_ROLE, k -> new HashSet<>()).addAll(paths);
} else {
result.computeIfAbsent(INGEST, k -> new HashSet<>()).addAll(paths);
result.computeIfAbsent(DiscoveryNodeRole.INGEST_ROLE, k -> new HashSet<>()).addAll(paths);
}
}
assertThat(result.size(), equalTo(pathsPerRole.size()));
for (DiscoveryNode.Role role : result.keySet()) {
for (DiscoveryNodeRole role : result.keySet()) {
assertThat("path are not the same for " + role, result.get(role), equalTo(pathsPerRole.get(role)));
}
} finally {

View File

@ -14,15 +14,15 @@ import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.common.xcontent.XContentHelper;
import org.elasticsearch.common.xcontent.XContentType;
import org.elasticsearch.test.AbstractStreamableTestCase;
import org.elasticsearch.xpack.core.ml.action.GetDatafeedsStatsAction.Response;
import org.elasticsearch.xpack.core.action.util.QueryPage;
import org.elasticsearch.xpack.core.ml.action.GetDatafeedsStatsAction.Response;
import org.elasticsearch.xpack.core.ml.datafeed.DatafeedConfig;
import org.elasticsearch.xpack.core.ml.datafeed.DatafeedState;
import java.io.IOException;
import java.net.InetAddress;
import java.util.ArrayList;
import java.util.EnumSet;
import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
@ -75,7 +75,7 @@ public class GetDatafeedStatsActionResponseTests extends AbstractStreamableTestC
TransportAddress transportAddress = new TransportAddress(TransportAddress.META_ADDRESS, 9000);
DiscoveryNode node = new DiscoveryNode("df-node-name", "df-node-id", transportAddress, attributes,
EnumSet.noneOf(DiscoveryNode.Role.class),
Collections.emptySet(),
Version.CURRENT);
Response.DatafeedStats stats = new Response.DatafeedStats("df-id", DatafeedState.STARTED, node, null);

View File

@ -13,6 +13,7 @@ import org.elasticsearch.client.Client;
import org.elasticsearch.cluster.ClusterState;
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.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.BytesStreamOutput;
@ -29,11 +30,11 @@ import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.xpack.core.XPackFeatureSet;
import org.elasticsearch.xpack.core.XPackFeatureSet.Usage;
import org.elasticsearch.xpack.core.XPackField;
import org.elasticsearch.xpack.core.action.util.QueryPage;
import org.elasticsearch.xpack.core.ml.MachineLearningFeatureSetUsage;
import org.elasticsearch.xpack.core.ml.MachineLearningField;
import org.elasticsearch.xpack.core.ml.action.GetDatafeedsStatsAction;
import org.elasticsearch.xpack.core.ml.action.GetJobsStatsAction;
import org.elasticsearch.xpack.core.action.util.QueryPage;
import org.elasticsearch.xpack.core.ml.datafeed.DatafeedConfig;
import org.elasticsearch.xpack.core.ml.datafeed.DatafeedState;
import org.elasticsearch.xpack.core.ml.job.config.AnalysisConfig;
@ -380,10 +381,10 @@ public class MachineLearningFeatureSetTests extends ESTestCase {
for (int i = 0; i < nodeCount; i++) {
Map<String, String> attrs = new HashMap<>();
attrs.put(MachineLearning.MAX_OPEN_JOBS_NODE_ATTR, Integer.toString(20));
Set<DiscoveryNode.Role> roles = new HashSet<>();
roles.add(DiscoveryNode.Role.DATA);
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNode.Role.INGEST);
Set<DiscoveryNodeRole> roles = new HashSet<>();
roles.add(DiscoveryNodeRole.DATA_ROLE);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
roles.add(DiscoveryNodeRole.INGEST_ROLE);
nodesBuilder.add(new DiscoveryNode("ml-feature-set-given-ml-node-" + i,
new TransportAddress(TransportAddress.META_ADDRESS, 9100 + i),
attrs,
@ -392,10 +393,10 @@ public class MachineLearningFeatureSetTests extends ESTestCase {
}
for (int i = 0; i < randomIntBetween(1, 3); i++) {
Map<String, String> attrs = new HashMap<>();
Set<DiscoveryNode.Role> roles = new HashSet<>();
roles.add(DiscoveryNode.Role.DATA);
roles.add(DiscoveryNode.Role.MASTER);
roles.add(DiscoveryNode.Role.INGEST);
Set<DiscoveryNodeRole> roles = new HashSet<>();
roles.add(DiscoveryNodeRole.DATA_ROLE);
roles.add(DiscoveryNodeRole.MASTER_ROLE);
roles.add(DiscoveryNodeRole.INGEST_ROLE);
nodesBuilder.add(new DiscoveryNode("ml-feature-set-given-non-ml-node-" + i,
new TransportAddress(TransportAddress.META_ADDRESS, 9300 + i),
attrs,

View File

@ -20,6 +20,7 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.health.ClusterHealthStatus;
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.cluster.routing.RecoverySource;
import org.elasticsearch.cluster.routing.ShardRouting;
@ -175,7 +176,7 @@ public class ClusterStatsMonitoringDocTests extends BaseMonitoringDocTestCase<Cl
randomAlphaOfLength(5),
new TransportAddress(TransportAddress.META_ADDRESS, 9301 + i),
randomBoolean() ? singletonMap("attr", randomAlphaOfLength(3)) : emptyMap,
singleton(randomFrom(DiscoveryNode.Role.values())),
singleton(randomFrom(DiscoveryNodeRole.BUILT_IN_ROLES)),
Version.CURRENT));
}
@ -201,7 +202,7 @@ public class ClusterStatsMonitoringDocTests extends BaseMonitoringDocTestCase<Cl
"_host_address",
transportAddress,
singletonMap("attr", "value"),
singleton(DiscoveryNode.Role.MASTER),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.V_6_0_0_beta1);
final ClusterState clusterState = ClusterState.builder(clusterName)
@ -440,10 +441,10 @@ public class ClusterStatsMonitoringDocTests extends BaseMonitoringDocTestCase<Cl
+ "\"nodes\":{"
+ "\"count\":{"
+ "\"total\":1,"
+ "\"data\":0,"
+ "\"coordinating_only\":0,"
+ "\"master\":1,"
+ "\"ingest\":0"
+ "\"data\":0,"
+ "\"ingest\":0,"
+ "\"master\":1"
+ "},"
+ "\"versions\":["
+ "\"6.0.0-alpha2\""
@ -585,7 +586,7 @@ public class ClusterStatsMonitoringDocTests extends BaseMonitoringDocTestCase<Cl
"_host_address",
new TransportAddress(TransportAddress.META_ADDRESS, 9300),
singletonMap("attr", "value"),
singleton(DiscoveryNode.Role.MASTER),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.CURRENT);
}

View File

@ -10,6 +10,7 @@ import org.elasticsearch.action.admin.indices.recovery.RecoveryResponse;
import org.elasticsearch.action.admin.indices.shards.IndicesShardStoresResponse;
import org.elasticsearch.action.support.DefaultShardOperationFailedException;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.routing.RecoverySource;
import org.elasticsearch.cluster.routing.ShardRouting;
import org.elasticsearch.cluster.routing.UnassignedInfo;
@ -78,7 +79,7 @@ public class IndexRecoveryMonitoringDocTests extends BaseMonitoringDocTestCase<I
"_host_address_0",
new TransportAddress(TransportAddress.META_ADDRESS, 9300),
singletonMap("attr", "value_0"),
singleton(DiscoveryNode.Role.MASTER),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.V_6_0_0_beta1);
final DiscoveryNode discoveryNodeOne = new DiscoveryNode("_node_1",
@ -88,7 +89,7 @@ public class IndexRecoveryMonitoringDocTests extends BaseMonitoringDocTestCase<I
"_host_address_1",
new TransportAddress(TransportAddress.META_ADDRESS, 9301),
singletonMap("attr", "value_1"),
singleton(DiscoveryNode.Role.DATA),
singleton(DiscoveryNodeRole.DATA_ROLE),
Version.V_6_0_0_alpha1);
final ShardId shardId = new ShardId("_index_a", "_uuid_a", 0);

View File

@ -7,6 +7,7 @@ package org.elasticsearch.xpack.monitoring.collector.ml;
import org.elasticsearch.Version;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.bytes.BytesReference;
import org.elasticsearch.common.transport.TransportAddress;
import org.elasticsearch.common.unit.TimeValue;
@ -85,7 +86,7 @@ public class JobStatsMonitoringDocTests extends BaseMonitoringDocTestCase<JobSta
"_host_address",
new TransportAddress(TransportAddress.META_ADDRESS, 9300),
singletonMap("attr", "value"),
singleton(DiscoveryNode.Role.MASTER),
singleton(DiscoveryNodeRole.MASTER_ROLE),
Version.V_6_0_0_beta1);
final ModelSizeStats modelStats = new ModelSizeStats.Builder("_model")

View File

@ -23,6 +23,7 @@ import org.elasticsearch.action.update.UpdateRequestBuilder;
import org.elasticsearch.client.Client;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.Strings;
@ -61,6 +62,7 @@ import org.junit.Before;
import org.junit.BeforeClass;
import javax.crypto.SecretKey;
import java.io.IOException;
import java.net.URLEncoder;
import java.nio.charset.StandardCharsets;
@ -813,7 +815,7 @@ public class TokenServiceTests extends ESTestCase {
final ClusterState currentState = clusterService.state();
final DiscoveryNodes.Builder discoBuilder = DiscoveryNodes.builder(currentState.getNodes());
final DiscoveryNode anotherDataNode = new DiscoveryNode("another_data_node#" + version, buildNewFakeTransportAddress(),
Collections.emptyMap(), Collections.singleton(DiscoveryNode.Role.DATA), version);
Collections.emptyMap(), Collections.singleton(DiscoveryNodeRole.DATA_ROLE), version);
discoBuilder.add(anotherDataNode);
final ClusterState.Builder newStateBuilder = ClusterState.builder(currentState);
newStateBuilder.nodes(discoBuilder);

View File

@ -8,6 +8,7 @@ package org.elasticsearch.xpack.security.transport;
import org.elasticsearch.Version;
import org.elasticsearch.action.support.PlainActionFuture;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.common.SuppressForbidden;
import org.elasticsearch.common.io.stream.OutputStreamStreamOutput;
import org.elasticsearch.common.settings.MockSecureSettings;
@ -45,6 +46,7 @@ import javax.net.ssl.SSLParameters;
import javax.net.ssl.SSLServerSocket;
import javax.net.ssl.SSLServerSocketFactory;
import javax.net.ssl.SSLSocket;
import java.io.IOException;
import java.io.UncheckedIOException;
import java.net.InetAddress;
@ -53,7 +55,6 @@ import java.net.SocketTimeoutException;
import java.net.UnknownHostException;
import java.nio.file.Path;
import java.util.Collections;
import java.util.EnumSet;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Locale;
@ -259,7 +260,7 @@ public abstract class AbstractSimpleSecurityTransportTestCase extends AbstractSi
HashMap<String, String> attributes = new HashMap<>();
attributes.put("server_name", sniIp);
DiscoveryNode node = new DiscoveryNode("server_node_id", new TransportAddress(serverAddress), attributes,
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
new Thread(() -> {
try {
@ -306,7 +307,7 @@ public abstract class AbstractSimpleSecurityTransportTestCase extends AbstractSi
HashMap<String, String> attributes = new HashMap<>();
attributes.put("server_name", sniIp);
DiscoveryNode node = new DiscoveryNode("server_node_id", new TransportAddress(serverAddress), attributes,
EnumSet.allOf(DiscoveryNode.Role.class), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
ConnectTransportException connectException = expectThrows(ConnectTransportException.class,
() -> serviceC.connectToNode(node, TestProfiles.LIGHT_PROFILE));

View File

@ -17,6 +17,7 @@ import org.elasticsearch.cluster.metadata.AliasOrIndex;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.RoutingNode;
@ -432,14 +433,14 @@ public class WatcherIndexingListenerTests extends ESTestCase {
DiscoveryNode node1 = new DiscoveryNode("node_1", ESTestCase.buildNewFakeTransportAddress(),
Collections.emptyMap(), new HashSet<>(Collections.singletonList(
randomFrom(DiscoveryNode.Role.INGEST, DiscoveryNode.Role.MASTER))),
randomFrom(DiscoveryNodeRole.INGEST_ROLE, DiscoveryNodeRole.MASTER_ROLE))),
Version.CURRENT);
DiscoveryNode node2 = new DiscoveryNode("node_2", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Collections.singletonList(DiscoveryNode.Role.DATA)), Version.CURRENT);
new HashSet<>(Collections.singletonList(DiscoveryNodeRole.DATA_ROLE)), Version.CURRENT);
DiscoveryNode node3 = new DiscoveryNode("node_3", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(Collections.singletonList(DiscoveryNode.Role.DATA)), Version.CURRENT);
new HashSet<>(Collections.singletonList(DiscoveryNodeRole.DATA_ROLE)), Version.CURRENT);
IndexMetaData.Builder indexMetaDataBuilder = createIndexBuilder(Watch.INDEX, 1 ,0);
@ -698,6 +699,6 @@ public class WatcherIndexingListenerTests extends ESTestCase {
private static DiscoveryNode newNode(String nodeId) {
return new DiscoveryNode(nodeId, ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.values())), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
}
}

View File

@ -17,6 +17,7 @@ import org.elasticsearch.cluster.metadata.IndexMetaData;
import org.elasticsearch.cluster.metadata.IndexTemplateMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.RoutingTable;
@ -327,13 +328,13 @@ public class WatcherLifeCycleServiceTests extends ESTestCase {
IndexRoutingTable.Builder indexRoutingTable = IndexRoutingTable.builder(index).addShard(shardRouting);
DiscoveryNode node1 = new DiscoveryNode("node_1", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(randomFrom(DiscoveryNode.Role.INGEST, DiscoveryNode.Role.MASTER))), Version.CURRENT);
new HashSet<>(asList(randomFrom(DiscoveryNodeRole.INGEST_ROLE, DiscoveryNodeRole.MASTER_ROLE))), Version.CURRENT);
DiscoveryNode node2 = new DiscoveryNode("node_2", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.DATA)), Version.CURRENT);
new HashSet<>(asList(DiscoveryNodeRole.DATA_ROLE)), Version.CURRENT);
DiscoveryNode node3 = new DiscoveryNode("node_3", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.DATA)), Version.CURRENT);
new HashSet<>(asList(DiscoveryNodeRole.DATA_ROLE)), Version.CURRENT);
IndexMetaData.Builder indexMetaDataBuilder = IndexMetaData.builder(Watch.INDEX)
.settings(Settings.builder()
@ -465,7 +466,8 @@ public class WatcherLifeCycleServiceTests extends ESTestCase {
}
public void testMasterOnlyNodeCanStart() {
List<DiscoveryNode.Role> roles = Collections.singletonList(randomFrom(DiscoveryNode.Role.MASTER, DiscoveryNode.Role.INGEST));
List<DiscoveryNodeRole> roles =
Collections.singletonList(randomFrom(DiscoveryNodeRole.MASTER_ROLE, DiscoveryNodeRole.INGEST_ROLE));
ClusterState state = ClusterState.builder(new ClusterName("my-cluster"))
.nodes(new DiscoveryNodes.Builder().masterNodeId("node_1").localNodeId("node_1")
.add(new DiscoveryNode("node_1", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
@ -596,6 +598,6 @@ public class WatcherLifeCycleServiceTests extends ESTestCase {
private static DiscoveryNode newNode(String nodeName, Version version) {
return new DiscoveryNode(nodeName, ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.values())), version);
DiscoveryNodeRole.BUILT_IN_ROLES, version);
}
}

View File

@ -28,6 +28,7 @@ import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.IndexMetaData;
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.cluster.routing.IndexRoutingTable;
import org.elasticsearch.cluster.routing.IndexShardRoutingTable;
@ -62,11 +63,9 @@ import org.mockito.ArgumentCaptor;
import java.time.ZoneOffset;
import java.time.ZonedDateTime;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
import java.util.concurrent.ExecutorService;
import static java.util.Arrays.asList;
import static org.hamcrest.Matchers.hasSize;
import static org.hamcrest.Matchers.is;
import static org.mockito.Matchers.any;
@ -277,6 +276,6 @@ public class WatcherServiceTests extends ESTestCase {
private static DiscoveryNode newNode() {
return new DiscoveryNode("node", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.values())), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
}
}

View File

@ -17,6 +17,7 @@ import org.elasticsearch.action.update.UpdateRequest;
import org.elasticsearch.action.update.UpdateResponse;
import org.elasticsearch.client.Client;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeRole;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.collect.Tuple;
import org.elasticsearch.common.settings.Settings;
@ -87,13 +88,11 @@ import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.concurrent.ArrayBlockingQueue;
import java.util.concurrent.atomic.AtomicBoolean;
import static java.util.Arrays.asList;
import static java.util.Collections.singletonMap;
import static org.elasticsearch.common.unit.TimeValue.timeValueSeconds;
import static org.elasticsearch.index.seqno.SequenceNumbers.UNASSIGNED_SEQ_NO;
@ -156,7 +155,7 @@ public class ExecutionServiceTests extends ESTestCase {
parser = mock(WatchParser.class);
DiscoveryNode discoveryNode = new DiscoveryNode("node_1", ESTestCase.buildNewFakeTransportAddress(), Collections.emptyMap(),
new HashSet<>(asList(DiscoveryNode.Role.values())), Version.CURRENT);
DiscoveryNodeRole.BUILT_IN_ROLES, Version.CURRENT);
ClusterService clusterService = mock(ClusterService.class);
when(clusterService.localNode()).thenReturn(discoveryNode);