NETWORKING: Make RemoteClusterConn. Lazy Resolve DNS (#32764)
* Lazy resolve DNS (i.e. `String` to `DiscoveryNode`) to not run into indefinitely caching lookup issues (provided the JVM dns cache is configured correctly as explained in https://www.elastic.co/guide/en/elasticsearch/reference/6.3/networkaddress-cache-ttl.html) * Changed `InetAddress` type to `String` for that higher up the stack * Passed down `Supplier<DiscoveryNode>` instead of outright `DiscoveryNode` from `RemoteClusterAware#buildRemoteClustersSeeds` on to lazy resolve DNS when the `DiscoveryNode` is actually used (could've also passed down the value of `clusterName = REMOTE_CLUSTERS_SEEDS.getNamespace(concreteSetting)` together with the `List<String>` of hosts, but this route seemed to introduce less duplication and resulted in a significantly smaller changeset). * Closes #28858
This commit is contained in:
parent
532d552ffd
commit
f82bb64feb
|
@ -18,6 +18,7 @@
|
||||||
*/
|
*/
|
||||||
package org.elasticsearch.transport;
|
package org.elasticsearch.transport;
|
||||||
|
|
||||||
|
import java.util.function.Supplier;
|
||||||
import org.elasticsearch.Version;
|
import org.elasticsearch.Version;
|
||||||
import org.elasticsearch.cluster.metadata.ClusterNameExpressionResolver;
|
import org.elasticsearch.cluster.metadata.ClusterNameExpressionResolver;
|
||||||
import org.elasticsearch.cluster.node.DiscoveryNode;
|
import org.elasticsearch.cluster.node.DiscoveryNode;
|
||||||
|
@ -48,9 +49,20 @@ public abstract class RemoteClusterAware extends AbstractComponent {
|
||||||
/**
|
/**
|
||||||
* A list of initial seed nodes to discover eligible nodes from the remote cluster
|
* A list of initial seed nodes to discover eligible nodes from the remote cluster
|
||||||
*/
|
*/
|
||||||
public static final Setting.AffixSetting<List<InetSocketAddress>> REMOTE_CLUSTERS_SEEDS = Setting.affixKeySetting("search.remote.",
|
public static final Setting.AffixSetting<List<String>> REMOTE_CLUSTERS_SEEDS = Setting.affixKeySetting(
|
||||||
"seeds", (key) -> Setting.listSetting(key, Collections.emptyList(), RemoteClusterAware::parseSeedAddress,
|
"search.remote.",
|
||||||
Setting.Property.NodeScope, Setting.Property.Dynamic));
|
"seeds",
|
||||||
|
key -> Setting.listSetting(
|
||||||
|
key, Collections.emptyList(),
|
||||||
|
s -> {
|
||||||
|
// validate seed address
|
||||||
|
parsePort(s);
|
||||||
|
return s;
|
||||||
|
},
|
||||||
|
Setting.Property.NodeScope,
|
||||||
|
Setting.Property.Dynamic
|
||||||
|
)
|
||||||
|
);
|
||||||
public static final char REMOTE_CLUSTER_INDEX_SEPARATOR = ':';
|
public static final char REMOTE_CLUSTER_INDEX_SEPARATOR = ':';
|
||||||
public static final String LOCAL_CLUSTER_GROUP_KEY = "";
|
public static final String LOCAL_CLUSTER_GROUP_KEY = "";
|
||||||
|
|
||||||
|
@ -65,18 +77,20 @@ public abstract class RemoteClusterAware extends AbstractComponent {
|
||||||
this.clusterNameResolver = new ClusterNameExpressionResolver(settings);
|
this.clusterNameResolver = new ClusterNameExpressionResolver(settings);
|
||||||
}
|
}
|
||||||
|
|
||||||
protected static Map<String, List<DiscoveryNode>> buildRemoteClustersSeeds(Settings settings) {
|
protected static Map<String, List<Supplier<DiscoveryNode>>> buildRemoteClustersSeeds(Settings settings) {
|
||||||
Stream<Setting<List<InetSocketAddress>>> allConcreteSettings = REMOTE_CLUSTERS_SEEDS.getAllConcreteSettings(settings);
|
Stream<Setting<List<String>>> allConcreteSettings = REMOTE_CLUSTERS_SEEDS.getAllConcreteSettings(settings);
|
||||||
return allConcreteSettings.collect(
|
return allConcreteSettings.collect(
|
||||||
Collectors.toMap(REMOTE_CLUSTERS_SEEDS::getNamespace, concreteSetting -> {
|
Collectors.toMap(REMOTE_CLUSTERS_SEEDS::getNamespace, concreteSetting -> {
|
||||||
String clusterName = REMOTE_CLUSTERS_SEEDS.getNamespace(concreteSetting);
|
String clusterName = REMOTE_CLUSTERS_SEEDS.getNamespace(concreteSetting);
|
||||||
List<DiscoveryNode> nodes = new ArrayList<>();
|
List<String> addresses = concreteSetting.get(settings);
|
||||||
for (InetSocketAddress address : concreteSetting.get(settings)) {
|
List<Supplier<DiscoveryNode>> nodes = new ArrayList<>(addresses.size());
|
||||||
TransportAddress transportAddress = new TransportAddress(address);
|
for (String address : addresses) {
|
||||||
DiscoveryNode node = new DiscoveryNode(clusterName + "#" + transportAddress.toString(),
|
nodes.add(() -> {
|
||||||
transportAddress,
|
TransportAddress transportAddress = new TransportAddress(RemoteClusterAware.parseSeedAddress(address));
|
||||||
Version.CURRENT.minimumCompatibilityVersion());
|
return new DiscoveryNode(clusterName + "#" + transportAddress.toString(),
|
||||||
nodes.add(node);
|
transportAddress,
|
||||||
|
Version.CURRENT.minimumCompatibilityVersion());
|
||||||
|
});
|
||||||
}
|
}
|
||||||
return nodes;
|
return nodes;
|
||||||
}));
|
}));
|
||||||
|
@ -128,7 +142,7 @@ public abstract class RemoteClusterAware extends AbstractComponent {
|
||||||
* Subclasses must implement this to receive information about updated cluster aliases. If the given address list is
|
* Subclasses must implement this to receive information about updated cluster aliases. If the given address list is
|
||||||
* empty the cluster alias is unregistered and should be removed.
|
* empty the cluster alias is unregistered and should be removed.
|
||||||
*/
|
*/
|
||||||
protected abstract void updateRemoteCluster(String clusterAlias, List<InetSocketAddress> addresses);
|
protected abstract void updateRemoteCluster(String clusterAlias, List<String> addresses);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Registers this instance to listen to updates on the cluster settings.
|
* Registers this instance to listen to updates on the cluster settings.
|
||||||
|
@ -138,29 +152,37 @@ public abstract class RemoteClusterAware extends AbstractComponent {
|
||||||
(namespace, value) -> {});
|
(namespace, value) -> {});
|
||||||
}
|
}
|
||||||
|
|
||||||
private static InetSocketAddress parseSeedAddress(String remoteHost) {
|
protected static InetSocketAddress parseSeedAddress(String remoteHost) {
|
||||||
int portSeparator = remoteHost.lastIndexOf(':'); // in case we have a IPv6 address ie. [::1]:9300
|
String host = remoteHost.substring(0, indexOfPortSeparator(remoteHost));
|
||||||
if (portSeparator == -1 || portSeparator == remoteHost.length()) {
|
|
||||||
throw new IllegalArgumentException("remote hosts need to be configured as [host:port], found [" + remoteHost + "] instead");
|
|
||||||
}
|
|
||||||
String host = remoteHost.substring(0, portSeparator);
|
|
||||||
InetAddress hostAddress;
|
InetAddress hostAddress;
|
||||||
try {
|
try {
|
||||||
hostAddress = InetAddress.getByName(host);
|
hostAddress = InetAddress.getByName(host);
|
||||||
} catch (UnknownHostException e) {
|
} catch (UnknownHostException e) {
|
||||||
throw new IllegalArgumentException("unknown host [" + host + "]", e);
|
throw new IllegalArgumentException("unknown host [" + host + "]", e);
|
||||||
}
|
}
|
||||||
|
return new InetSocketAddress(hostAddress, parsePort(remoteHost));
|
||||||
|
}
|
||||||
|
|
||||||
|
private static int parsePort(String remoteHost) {
|
||||||
try {
|
try {
|
||||||
int port = Integer.valueOf(remoteHost.substring(portSeparator + 1));
|
int port = Integer.valueOf(remoteHost.substring(indexOfPortSeparator(remoteHost) + 1));
|
||||||
if (port <= 0) {
|
if (port <= 0) {
|
||||||
throw new IllegalArgumentException("port number must be > 0 but was: [" + port + "]");
|
throw new IllegalArgumentException("port number must be > 0 but was: [" + port + "]");
|
||||||
}
|
}
|
||||||
return new InetSocketAddress(hostAddress, port);
|
return port;
|
||||||
} catch (NumberFormatException e) {
|
} catch (NumberFormatException e) {
|
||||||
throw new IllegalArgumentException("port must be a number", e);
|
throw new IllegalArgumentException("failed to parse port", e);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
private static int indexOfPortSeparator(String remoteHost) {
|
||||||
|
int portSeparator = remoteHost.lastIndexOf(':'); // in case we have a IPv6 address ie. [::1]:9300
|
||||||
|
if (portSeparator == -1 || portSeparator == remoteHost.length()) {
|
||||||
|
throw new IllegalArgumentException("remote hosts need to be configured as [host:port], found [" + remoteHost + "] instead");
|
||||||
|
}
|
||||||
|
return portSeparator;
|
||||||
|
}
|
||||||
|
|
||||||
public static String buildRemoteIndexName(String clusterAlias, String indexName) {
|
public static String buildRemoteIndexName(String clusterAlias, String indexName) {
|
||||||
return clusterAlias != null ? clusterAlias + REMOTE_CLUSTER_INDEX_SEPARATOR + indexName : indexName;
|
return clusterAlias != null ? clusterAlias + REMOTE_CLUSTER_INDEX_SEPARATOR + indexName : indexName;
|
||||||
}
|
}
|
||||||
|
|
|
@ -18,6 +18,7 @@
|
||||||
*/
|
*/
|
||||||
package org.elasticsearch.transport;
|
package org.elasticsearch.transport;
|
||||||
|
|
||||||
|
import java.util.function.Supplier;
|
||||||
import org.apache.logging.log4j.message.ParameterizedMessage;
|
import org.apache.logging.log4j.message.ParameterizedMessage;
|
||||||
import org.apache.lucene.store.AlreadyClosedException;
|
import org.apache.lucene.store.AlreadyClosedException;
|
||||||
import org.apache.lucene.util.SetOnce;
|
import org.apache.lucene.util.SetOnce;
|
||||||
|
@ -84,7 +85,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
private final String clusterAlias;
|
private final String clusterAlias;
|
||||||
private final int maxNumRemoteConnections;
|
private final int maxNumRemoteConnections;
|
||||||
private final Predicate<DiscoveryNode> nodePredicate;
|
private final Predicate<DiscoveryNode> nodePredicate;
|
||||||
private volatile List<DiscoveryNode> seedNodes;
|
private volatile List<Supplier<DiscoveryNode>> seedNodes;
|
||||||
private volatile boolean skipUnavailable;
|
private volatile boolean skipUnavailable;
|
||||||
private final ConnectHandler connectHandler;
|
private final ConnectHandler connectHandler;
|
||||||
private SetOnce<ClusterName> remoteClusterName = new SetOnce<>();
|
private SetOnce<ClusterName> remoteClusterName = new SetOnce<>();
|
||||||
|
@ -99,7 +100,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
* @param maxNumRemoteConnections the maximum number of connections to the remote cluster
|
* @param maxNumRemoteConnections the maximum number of connections to the remote cluster
|
||||||
* @param nodePredicate a predicate to filter eligible remote nodes to connect to
|
* @param nodePredicate a predicate to filter eligible remote nodes to connect to
|
||||||
*/
|
*/
|
||||||
RemoteClusterConnection(Settings settings, String clusterAlias, List<DiscoveryNode> seedNodes,
|
RemoteClusterConnection(Settings settings, String clusterAlias, List<Supplier<DiscoveryNode>> seedNodes,
|
||||||
TransportService transportService, int maxNumRemoteConnections, Predicate<DiscoveryNode> nodePredicate) {
|
TransportService transportService, int maxNumRemoteConnections, Predicate<DiscoveryNode> nodePredicate) {
|
||||||
super(settings);
|
super(settings);
|
||||||
this.localClusterName = ClusterName.CLUSTER_NAME_SETTING.get(settings);
|
this.localClusterName = ClusterName.CLUSTER_NAME_SETTING.get(settings);
|
||||||
|
@ -127,7 +128,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
/**
|
/**
|
||||||
* Updates the list of seed nodes for this cluster connection
|
* Updates the list of seed nodes for this cluster connection
|
||||||
*/
|
*/
|
||||||
synchronized void updateSeedNodes(List<DiscoveryNode> seedNodes, ActionListener<Void> connectListener) {
|
synchronized void updateSeedNodes(List<Supplier<DiscoveryNode>> seedNodes, ActionListener<Void> connectListener) {
|
||||||
this.seedNodes = Collections.unmodifiableList(new ArrayList<>(seedNodes));
|
this.seedNodes = Collections.unmodifiableList(new ArrayList<>(seedNodes));
|
||||||
connectHandler.connect(connectListener);
|
connectHandler.connect(connectListener);
|
||||||
}
|
}
|
||||||
|
@ -456,7 +457,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
});
|
});
|
||||||
}
|
}
|
||||||
|
|
||||||
void collectRemoteNodes(Iterator<DiscoveryNode> seedNodes,
|
private void collectRemoteNodes(Iterator<Supplier<DiscoveryNode>> seedNodes,
|
||||||
final TransportService transportService, ActionListener<Void> listener) {
|
final TransportService transportService, ActionListener<Void> listener) {
|
||||||
if (Thread.currentThread().isInterrupted()) {
|
if (Thread.currentThread().isInterrupted()) {
|
||||||
listener.onFailure(new InterruptedException("remote connect thread got interrupted"));
|
listener.onFailure(new InterruptedException("remote connect thread got interrupted"));
|
||||||
|
@ -464,7 +465,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
try {
|
try {
|
||||||
if (seedNodes.hasNext()) {
|
if (seedNodes.hasNext()) {
|
||||||
cancellableThreads.executeIO(() -> {
|
cancellableThreads.executeIO(() -> {
|
||||||
final DiscoveryNode seedNode = seedNodes.next();
|
final DiscoveryNode seedNode = seedNodes.next().get();
|
||||||
final TransportService.HandshakeResponse handshakeResponse;
|
final TransportService.HandshakeResponse handshakeResponse;
|
||||||
Transport.Connection connection = transportService.openConnection(seedNode,
|
Transport.Connection connection = transportService.openConnection(seedNode,
|
||||||
ConnectionProfile.buildSingleChannelProfile(TransportRequestOptions.Type.REG, null, null));
|
ConnectionProfile.buildSingleChannelProfile(TransportRequestOptions.Type.REG, null, null));
|
||||||
|
@ -554,11 +555,11 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
private final TransportService transportService;
|
private final TransportService transportService;
|
||||||
private final Transport.Connection connection;
|
private final Transport.Connection connection;
|
||||||
private final ActionListener<Void> listener;
|
private final ActionListener<Void> listener;
|
||||||
private final Iterator<DiscoveryNode> seedNodes;
|
private final Iterator<Supplier<DiscoveryNode>> seedNodes;
|
||||||
private final CancellableThreads cancellableThreads;
|
private final CancellableThreads cancellableThreads;
|
||||||
|
|
||||||
SniffClusterStateResponseHandler(TransportService transportService, Transport.Connection connection,
|
SniffClusterStateResponseHandler(TransportService transportService, Transport.Connection connection,
|
||||||
ActionListener<Void> listener, Iterator<DiscoveryNode> seedNodes,
|
ActionListener<Void> listener, Iterator<Supplier<DiscoveryNode>> seedNodes,
|
||||||
CancellableThreads cancellableThreads) {
|
CancellableThreads cancellableThreads) {
|
||||||
this.transportService = transportService;
|
this.transportService = transportService;
|
||||||
this.connection = connection;
|
this.connection = connection;
|
||||||
|
@ -651,7 +652,7 @@ final class RemoteClusterConnection extends AbstractComponent implements Transpo
|
||||||
* Get the information about remote nodes to be rendered on {@code _remote/info} requests.
|
* Get the information about remote nodes to be rendered on {@code _remote/info} requests.
|
||||||
*/
|
*/
|
||||||
public RemoteConnectionInfo getConnectionInfo() {
|
public RemoteConnectionInfo getConnectionInfo() {
|
||||||
List<TransportAddress> seedNodeAddresses = seedNodes.stream().map(DiscoveryNode::getAddress).collect(Collectors.toList());
|
List<TransportAddress> seedNodeAddresses = seedNodes.stream().map(node -> node.get().getAddress()).collect(Collectors.toList());
|
||||||
TimeValue initialConnectionTimeout = RemoteClusterService.REMOTE_INITIAL_CONNECTION_TIMEOUT_SETTING.get(settings);
|
TimeValue initialConnectionTimeout = RemoteClusterService.REMOTE_INITIAL_CONNECTION_TIMEOUT_SETTING.get(settings);
|
||||||
return new RemoteConnectionInfo(clusterAlias, seedNodeAddresses, maxNumRemoteConnections, connectedNodes.size(),
|
return new RemoteConnectionInfo(clusterAlias, seedNodeAddresses, maxNumRemoteConnections, connectedNodes.size(),
|
||||||
initialConnectionTimeout, skipUnavailable);
|
initialConnectionTimeout, skipUnavailable);
|
||||||
|
|
|
@ -18,6 +18,7 @@
|
||||||
*/
|
*/
|
||||||
package org.elasticsearch.transport;
|
package org.elasticsearch.transport;
|
||||||
|
|
||||||
|
import java.util.function.Supplier;
|
||||||
import org.elasticsearch.Version;
|
import org.elasticsearch.Version;
|
||||||
import org.elasticsearch.action.ActionListener;
|
import org.elasticsearch.action.ActionListener;
|
||||||
import org.elasticsearch.action.OriginalIndices;
|
import org.elasticsearch.action.OriginalIndices;
|
||||||
|
@ -40,7 +41,6 @@ import org.elasticsearch.threadpool.ThreadPool;
|
||||||
|
|
||||||
import java.io.Closeable;
|
import java.io.Closeable;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.net.InetSocketAddress;
|
|
||||||
import java.util.Collections;
|
import java.util.Collections;
|
||||||
import java.util.HashMap;
|
import java.util.HashMap;
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
|
@ -115,7 +115,8 @@ public final class RemoteClusterService extends RemoteClusterAware implements Cl
|
||||||
* @param seeds a cluster alias to discovery node mapping representing the remote clusters seeds nodes
|
* @param seeds a cluster alias to discovery node mapping representing the remote clusters seeds nodes
|
||||||
* @param connectionListener a listener invoked once every configured cluster has been connected to
|
* @param connectionListener a listener invoked once every configured cluster has been connected to
|
||||||
*/
|
*/
|
||||||
private synchronized void updateRemoteClusters(Map<String, List<DiscoveryNode>> seeds, ActionListener<Void> connectionListener) {
|
private synchronized void updateRemoteClusters(Map<String, List<Supplier<DiscoveryNode>>> seeds,
|
||||||
|
ActionListener<Void> connectionListener) {
|
||||||
if (seeds.containsKey(LOCAL_CLUSTER_GROUP_KEY)) {
|
if (seeds.containsKey(LOCAL_CLUSTER_GROUP_KEY)) {
|
||||||
throw new IllegalArgumentException("remote clusters must not have the empty string as its key");
|
throw new IllegalArgumentException("remote clusters must not have the empty string as its key");
|
||||||
}
|
}
|
||||||
|
@ -125,7 +126,7 @@ public final class RemoteClusterService extends RemoteClusterAware implements Cl
|
||||||
} else {
|
} else {
|
||||||
CountDown countDown = new CountDown(seeds.size());
|
CountDown countDown = new CountDown(seeds.size());
|
||||||
remoteClusters.putAll(this.remoteClusters);
|
remoteClusters.putAll(this.remoteClusters);
|
||||||
for (Map.Entry<String, List<DiscoveryNode>> entry : seeds.entrySet()) {
|
for (Map.Entry<String, List<Supplier<DiscoveryNode>>> entry : seeds.entrySet()) {
|
||||||
RemoteClusterConnection remote = this.remoteClusters.get(entry.getKey());
|
RemoteClusterConnection remote = this.remoteClusters.get(entry.getKey());
|
||||||
if (entry.getValue().isEmpty()) { // with no seed nodes we just remove the connection
|
if (entry.getValue().isEmpty()) { // with no seed nodes we just remove the connection
|
||||||
try {
|
try {
|
||||||
|
@ -310,16 +311,17 @@ public final class RemoteClusterService extends RemoteClusterAware implements Cl
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
protected void updateRemoteCluster(String clusterAlias, List<InetSocketAddress> addresses) {
|
@Override
|
||||||
|
protected void updateRemoteCluster(String clusterAlias, List<String> addresses) {
|
||||||
updateRemoteCluster(clusterAlias, addresses, ActionListener.wrap((x) -> {}, (x) -> {}));
|
updateRemoteCluster(clusterAlias, addresses, ActionListener.wrap((x) -> {}, (x) -> {}));
|
||||||
}
|
}
|
||||||
|
|
||||||
void updateRemoteCluster(
|
void updateRemoteCluster(
|
||||||
final String clusterAlias,
|
final String clusterAlias,
|
||||||
final List<InetSocketAddress> addresses,
|
final List<String> addresses,
|
||||||
final ActionListener<Void> connectionListener) {
|
final ActionListener<Void> connectionListener) {
|
||||||
final List<DiscoveryNode> nodes = addresses.stream().map(address -> {
|
final List<Supplier<DiscoveryNode>> nodes = addresses.stream().<Supplier<DiscoveryNode>>map(address -> () -> {
|
||||||
final TransportAddress transportAddress = new TransportAddress(address);
|
final TransportAddress transportAddress = new TransportAddress(RemoteClusterAware.parseSeedAddress(address));
|
||||||
final String id = clusterAlias + "#" + transportAddress.toString();
|
final String id = clusterAlias + "#" + transportAddress.toString();
|
||||||
final Version version = Version.CURRENT.minimumCompatibilityVersion();
|
final Version version = Version.CURRENT.minimumCompatibilityVersion();
|
||||||
return new DiscoveryNode(id, transportAddress, version);
|
return new DiscoveryNode(id, transportAddress, version);
|
||||||
|
@ -334,7 +336,7 @@ public final class RemoteClusterService extends RemoteClusterAware implements Cl
|
||||||
void initializeRemoteClusters() {
|
void initializeRemoteClusters() {
|
||||||
final TimeValue timeValue = REMOTE_INITIAL_CONNECTION_TIMEOUT_SETTING.get(settings);
|
final TimeValue timeValue = REMOTE_INITIAL_CONNECTION_TIMEOUT_SETTING.get(settings);
|
||||||
final PlainActionFuture<Void> future = new PlainActionFuture<>();
|
final PlainActionFuture<Void> future = new PlainActionFuture<>();
|
||||||
Map<String, List<DiscoveryNode>> seeds = RemoteClusterAware.buildRemoteClustersSeeds(settings);
|
Map<String, List<Supplier<DiscoveryNode>>> seeds = RemoteClusterAware.buildRemoteClustersSeeds(settings);
|
||||||
updateRemoteClusters(seeds, future);
|
updateRemoteClusters(seeds, future);
|
||||||
try {
|
try {
|
||||||
future.get(timeValue.millis(), TimeUnit.MILLISECONDS);
|
future.get(timeValue.millis(), TimeUnit.MILLISECONDS);
|
||||||
|
|
|
@ -18,6 +18,7 @@
|
||||||
*/
|
*/
|
||||||
package org.elasticsearch.transport;
|
package org.elasticsearch.transport;
|
||||||
|
|
||||||
|
import java.util.function.Supplier;
|
||||||
import org.apache.lucene.store.AlreadyClosedException;
|
import org.apache.lucene.store.AlreadyClosedException;
|
||||||
import org.elasticsearch.Version;
|
import org.elasticsearch.Version;
|
||||||
import org.elasticsearch.action.ActionListener;
|
import org.elasticsearch.action.ActionListener;
|
||||||
|
@ -158,8 +159,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
|
@ -198,8 +199,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
|
@ -254,8 +255,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
|
@ -276,7 +277,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
||||||
knownNodes.add(incompatibleTransport.getLocalDiscoNode());
|
knownNodes.add(incompatibleTransport.getLocalDiscoNode());
|
||||||
Collections.shuffle(knownNodes, random());
|
Collections.shuffle(knownNodes, random());
|
||||||
List<DiscoveryNode> seedNodes = Arrays.asList(incompatibleSeedNode, seedNode);
|
List<Supplier<DiscoveryNode>> seedNodes = Arrays.asList(() -> incompatibleSeedNode, () -> seedNode);
|
||||||
Collections.shuffle(seedNodes, random());
|
Collections.shuffle(seedNodes, random());
|
||||||
|
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
@ -310,8 +311,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertFalse(service.nodeConnected(spareNode));
|
assertFalse(service.nodeConnected(spareNode));
|
||||||
|
@ -359,8 +360,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> n.equals(rejectedNode) == false)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> n.equals(rejectedNode) == false)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
if (rejectedNode.equals(seedNode)) {
|
if (rejectedNode.equals(seedNode)) {
|
||||||
assertFalse(service.nodeConnected(seedNode));
|
assertFalse(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
|
@ -374,7 +375,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
private void updateSeedNodes(RemoteClusterConnection connection, List<DiscoveryNode> seedNodes) throws Exception {
|
private void updateSeedNodes(RemoteClusterConnection connection, List<Supplier<DiscoveryNode>> seedNodes) throws Exception {
|
||||||
CountDownLatch latch = new CountDownLatch(1);
|
CountDownLatch latch = new CountDownLatch(1);
|
||||||
AtomicReference<Exception> exceptionAtomicReference = new AtomicReference<>();
|
AtomicReference<Exception> exceptionAtomicReference = new AtomicReference<>();
|
||||||
ActionListener<Void> listener = ActionListener.wrap(x -> latch.countDown(), x -> {
|
ActionListener<Void> listener = ActionListener.wrap(x -> latch.countDown(), x -> {
|
||||||
|
@ -398,8 +399,8 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
expectThrows(Exception.class, () -> updateSeedNodes(connection, Arrays.asList(seedNode)));
|
expectThrows(Exception.class, () -> updateSeedNodes(connection, Arrays.asList(() -> seedNode)));
|
||||||
assertFalse(service.nodeConnected(seedNode));
|
assertFalse(service.nodeConnected(seedNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
}
|
}
|
||||||
|
@ -461,7 +462,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
connection.addConnectedNode(seedNode);
|
connection.addConnectedNode(seedNode);
|
||||||
for (DiscoveryNode node : knownNodes) {
|
for (DiscoveryNode node : knownNodes) {
|
||||||
final Transport.Connection transportConnection = connection.getConnection(node);
|
final Transport.Connection transportConnection = connection.getConnection(node);
|
||||||
|
@ -504,7 +505,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
CountDownLatch listenerCalled = new CountDownLatch(1);
|
CountDownLatch listenerCalled = new CountDownLatch(1);
|
||||||
AtomicReference<Exception> exceptionReference = new AtomicReference<>();
|
AtomicReference<Exception> exceptionReference = new AtomicReference<>();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
ActionListener<Void> listener = ActionListener.wrap(x -> {
|
ActionListener<Void> listener = ActionListener.wrap(x -> {
|
||||||
listenerCalled.countDown();
|
listenerCalled.countDown();
|
||||||
fail("expected exception");
|
fail("expected exception");
|
||||||
|
@ -512,7 +513,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
exceptionReference.set(x);
|
exceptionReference.set(x);
|
||||||
listenerCalled.countDown();
|
listenerCalled.countDown();
|
||||||
});
|
});
|
||||||
connection.updateSeedNodes(Arrays.asList(seedNode), listener);
|
connection.updateSeedNodes(Arrays.asList(() -> seedNode), listener);
|
||||||
acceptedLatch.await();
|
acceptedLatch.await();
|
||||||
connection.close(); // now close it, this should trigger an interrupt on the socket and we can move on
|
connection.close(); // now close it, this should trigger an interrupt on the socket and we can move on
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
|
@ -539,7 +540,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
List<DiscoveryNode> nodes = Collections.singletonList(seedNode);
|
List<Supplier<DiscoveryNode>> nodes = Collections.singletonList(() -> seedNode);
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
nodes, service, Integer.MAX_VALUE, n -> true)) {
|
nodes, service, Integer.MAX_VALUE, n -> true)) {
|
||||||
if (randomBoolean()) {
|
if (randomBoolean()) {
|
||||||
|
@ -579,7 +580,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
List<DiscoveryNode> nodes = Collections.singletonList(seedNode);
|
List<Supplier<DiscoveryNode>> nodes = Collections.singletonList(() -> seedNode);
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
nodes, service, Integer.MAX_VALUE, n -> true)) {
|
nodes, service, Integer.MAX_VALUE, n -> true)) {
|
||||||
SearchRequest request = new SearchRequest("test-index");
|
SearchRequest request = new SearchRequest("test-index");
|
||||||
|
@ -635,7 +636,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Collections.singletonList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Collections.singletonList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
|
|
||||||
SearchRequest request = new SearchRequest("test-index");
|
SearchRequest request = new SearchRequest("test-index");
|
||||||
ClusterSearchShardsRequest searchShardsRequest = new ClusterSearchShardsRequest("test-index")
|
ClusterSearchShardsRequest searchShardsRequest = new ClusterSearchShardsRequest("test-index")
|
||||||
|
@ -738,7 +739,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
||||||
knownNodes.add(seedTransport1.getLocalDiscoNode());
|
knownNodes.add(seedTransport1.getLocalDiscoNode());
|
||||||
Collections.shuffle(knownNodes, random());
|
Collections.shuffle(knownNodes, random());
|
||||||
List<DiscoveryNode> seedNodes = Arrays.asList(seedNode1, seedNode);
|
List<Supplier<DiscoveryNode>> seedNodes = Arrays.asList(() -> seedNode1, () -> seedNode);
|
||||||
Collections.shuffle(seedNodes, random());
|
Collections.shuffle(seedNodes, random());
|
||||||
|
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
@ -816,7 +817,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
||||||
knownNodes.add(seedTransport1.getLocalDiscoNode());
|
knownNodes.add(seedTransport1.getLocalDiscoNode());
|
||||||
Collections.shuffle(knownNodes, random());
|
Collections.shuffle(knownNodes, random());
|
||||||
List<DiscoveryNode> seedNodes = Arrays.asList(seedNode1, seedNode);
|
List<Supplier<DiscoveryNode>> seedNodes = Arrays.asList(() -> seedNode1, () -> seedNode);
|
||||||
Collections.shuffle(seedNodes, random());
|
Collections.shuffle(seedNodes, random());
|
||||||
|
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
@ -904,7 +905,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
knownNodes.add(transport3.getLocalDiscoNode());
|
knownNodes.add(transport3.getLocalDiscoNode());
|
||||||
knownNodes.add(transport2.getLocalDiscoNode());
|
knownNodes.add(transport2.getLocalDiscoNode());
|
||||||
Collections.shuffle(knownNodes, random());
|
Collections.shuffle(knownNodes, random());
|
||||||
List<DiscoveryNode> seedNodes = Arrays.asList(node3, node1, node2);
|
List<Supplier<DiscoveryNode>> seedNodes = Arrays.asList(() -> node3, () -> node1, () -> node2);
|
||||||
Collections.shuffle(seedNodes, random());
|
Collections.shuffle(seedNodes, random());
|
||||||
|
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
@ -1059,7 +1060,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
assertFalse(service.nodeConnected(seedNode));
|
assertFalse(service.nodeConnected(seedNode));
|
||||||
assertFalse(service.nodeConnected(discoverableNode));
|
assertFalse(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
|
@ -1108,9 +1109,9 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList(() -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
if (randomBoolean()) {
|
if (randomBoolean()) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
}
|
}
|
||||||
CountDownLatch responseLatch = new CountDownLatch(1);
|
CountDownLatch responseLatch = new CountDownLatch(1);
|
||||||
AtomicReference<Function<String, DiscoveryNode>> reference = new AtomicReference<>();
|
AtomicReference<Function<String, DiscoveryNode>> reference = new AtomicReference<>();
|
||||||
|
@ -1142,14 +1143,14 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
List<MockTransportService> discoverableTransports = new CopyOnWriteArrayList<>();
|
List<MockTransportService> discoverableTransports = new CopyOnWriteArrayList<>();
|
||||||
try {
|
try {
|
||||||
final int numDiscoverableNodes = randomIntBetween(5, 20);
|
final int numDiscoverableNodes = randomIntBetween(5, 20);
|
||||||
List<DiscoveryNode> discoverableNodes = new ArrayList<>(numDiscoverableNodes);
|
List<Supplier<DiscoveryNode>> discoverableNodes = new ArrayList<>(numDiscoverableNodes);
|
||||||
for (int i = 0; i < numDiscoverableNodes; i++) {
|
for (int i = 0; i < numDiscoverableNodes; i++ ) {
|
||||||
MockTransportService transportService = startTransport("discoverable_node" + i, knownNodes, Version.CURRENT);
|
MockTransportService transportService = startTransport("discoverable_node" + i, knownNodes, Version.CURRENT);
|
||||||
discoverableNodes.add(transportService.getLocalDiscoNode());
|
discoverableNodes.add(transportService::getLocalDiscoNode);
|
||||||
discoverableTransports.add(transportService);
|
discoverableTransports.add(transportService);
|
||||||
}
|
}
|
||||||
|
|
||||||
List<DiscoveryNode> seedNodes = randomSubsetOf(discoverableNodes);
|
List<Supplier<DiscoveryNode>> seedNodes = randomSubsetOf(discoverableNodes);
|
||||||
Collections.shuffle(seedNodes, random());
|
Collections.shuffle(seedNodes, random());
|
||||||
|
|
||||||
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
@ -1198,7 +1199,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
discoverableTransports.add(transportService);
|
discoverableTransports.add(transportService);
|
||||||
connection.addConnectedNode(transportService.getLocalDiscoNode());
|
connection.addConnectedNode(transportService.getLocalDiscoNode());
|
||||||
} else {
|
} else {
|
||||||
DiscoveryNode node = randomFrom(discoverableNodes);
|
DiscoveryNode node = randomFrom(discoverableNodes).get();
|
||||||
connection.onNodeDisconnected(node);
|
connection.onNodeDisconnected(node);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
@ -1246,12 +1247,13 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Arrays.asList(seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Arrays.asList( () -> seedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
updateSeedNodes(connection, Arrays.asList(seedNode));
|
updateSeedNodes(connection, Arrays.asList(() -> seedNode));
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
List<DiscoveryNode> discoveryNodes = Arrays.asList(otherClusterTransport.getLocalDiscoNode(), seedNode);
|
List<Supplier<DiscoveryNode>> discoveryNodes =
|
||||||
|
Arrays.asList(() -> otherClusterTransport.getLocalDiscoNode(), () -> seedNode);
|
||||||
Collections.shuffle(discoveryNodes, random());
|
Collections.shuffle(discoveryNodes, random());
|
||||||
updateSeedNodes(connection, discoveryNodes);
|
updateSeedNodes(connection, discoveryNodes);
|
||||||
assertTrue(service.nodeConnected(seedNode));
|
assertTrue(service.nodeConnected(seedNode));
|
||||||
|
@ -1262,7 +1264,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
assertTrue(service.nodeConnected(discoverableNode));
|
assertTrue(service.nodeConnected(discoverableNode));
|
||||||
assertTrue(connection.assertNoRunningConnections());
|
assertTrue(connection.assertNoRunningConnections());
|
||||||
IllegalStateException illegalStateException = expectThrows(IllegalStateException.class, () ->
|
IllegalStateException illegalStateException = expectThrows(IllegalStateException.class, () ->
|
||||||
updateSeedNodes(connection, Arrays.asList(otherClusterTransport.getLocalDiscoNode())));
|
updateSeedNodes(connection, Arrays.asList(() -> otherClusterTransport.getLocalDiscoNode())));
|
||||||
assertThat(illegalStateException.getMessage(),
|
assertThat(illegalStateException.getMessage(),
|
||||||
startsWith("handshake failed, mismatched cluster name [Cluster [otherCluster]]" +
|
startsWith("handshake failed, mismatched cluster name [Cluster [otherCluster]]" +
|
||||||
" - {other_cluster_discoverable_node}"));
|
" - {other_cluster_discoverable_node}"));
|
||||||
|
@ -1325,7 +1327,7 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
service.start();
|
service.start();
|
||||||
service.acceptIncomingRequests();
|
service.acceptIncomingRequests();
|
||||||
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
Collections.singletonList(connectedNode), service, Integer.MAX_VALUE, n -> true)) {
|
Collections.singletonList(() -> connectedNode), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
connection.addConnectedNode(connectedNode);
|
connection.addConnectedNode(connectedNode);
|
||||||
for (int i = 0; i < 10; i++) {
|
for (int i = 0; i < 10; i++) {
|
||||||
//always a direct connection as the remote node is already connected
|
//always a direct connection as the remote node is already connected
|
||||||
|
@ -1348,4 +1350,34 @@ public class RemoteClusterConnectionTests extends ESTestCase {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public void testLazyResolveTransportAddress() throws Exception {
|
||||||
|
List<DiscoveryNode> knownNodes = new CopyOnWriteArrayList<>();
|
||||||
|
try (MockTransportService seedTransport = startTransport("seed_node", knownNodes, Version.CURRENT);
|
||||||
|
MockTransportService discoverableTransport = startTransport("discoverable_node", knownNodes, Version.CURRENT)) {
|
||||||
|
DiscoveryNode seedNode = seedTransport.getLocalDiscoNode();
|
||||||
|
knownNodes.add(seedTransport.getLocalDiscoNode());
|
||||||
|
knownNodes.add(discoverableTransport.getLocalDiscoNode());
|
||||||
|
Collections.shuffle(knownNodes, random());
|
||||||
|
|
||||||
|
try (MockTransportService service = MockTransportService.createNewService(Settings.EMPTY, Version.CURRENT, threadPool, null)) {
|
||||||
|
service.start();
|
||||||
|
service.acceptIncomingRequests();
|
||||||
|
CountDownLatch multipleResolveLatch = new CountDownLatch(2);
|
||||||
|
Supplier<DiscoveryNode> seedSupplier = () -> {
|
||||||
|
multipleResolveLatch.countDown();
|
||||||
|
return seedNode;
|
||||||
|
};
|
||||||
|
try (RemoteClusterConnection connection = new RemoteClusterConnection(Settings.EMPTY, "test-cluster",
|
||||||
|
Arrays.asList(seedSupplier), service, Integer.MAX_VALUE, n -> true)) {
|
||||||
|
updateSeedNodes(connection, Arrays.asList(seedSupplier));
|
||||||
|
// Closing connections leads to RemoteClusterConnection.ConnectHandler.collectRemoteNodes
|
||||||
|
// being called again so we try to resolve the same seed node's host twice
|
||||||
|
discoverableTransport.close();
|
||||||
|
seedTransport.close();
|
||||||
|
assertTrue(multipleResolveLatch.await(30L, TimeUnit.SECONDS));
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -18,6 +18,7 @@
|
||||||
*/
|
*/
|
||||||
package org.elasticsearch.transport;
|
package org.elasticsearch.transport;
|
||||||
|
|
||||||
|
import java.util.function.Supplier;
|
||||||
import org.elasticsearch.Version;
|
import org.elasticsearch.Version;
|
||||||
import org.elasticsearch.action.ActionListener;
|
import org.elasticsearch.action.ActionListener;
|
||||||
import org.elasticsearch.action.LatchedActionListener;
|
import org.elasticsearch.action.LatchedActionListener;
|
||||||
|
@ -103,10 +104,19 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
.put("search.remote.foo.seeds", "192.168.0.1").build();
|
.put("search.remote.foo.seeds", "192.168.0.1").build();
|
||||||
expectThrows(IllegalArgumentException.class, () ->
|
expectThrows(IllegalArgumentException.class, () ->
|
||||||
RemoteClusterAware.REMOTE_CLUSTERS_SEEDS.getAllConcreteSettings(brokenSettings).forEach(setting -> setting.get(brokenSettings)));
|
RemoteClusterAware.REMOTE_CLUSTERS_SEEDS.getAllConcreteSettings(brokenSettings).forEach(setting -> setting.get(brokenSettings)));
|
||||||
|
|
||||||
|
Settings brokenPortSettings = Settings.builder()
|
||||||
|
.put("search.remote.foo.seeds", "192.168.0.1:123456789123456789").build();
|
||||||
|
Exception e = expectThrows(
|
||||||
|
IllegalArgumentException.class,
|
||||||
|
() -> RemoteClusterAware.REMOTE_CLUSTERS_SEEDS.getAllConcreteSettings(brokenSettings)
|
||||||
|
.forEach(setting -> setting.get(brokenPortSettings))
|
||||||
|
);
|
||||||
|
assertEquals("failed to parse port", e.getMessage());
|
||||||
}
|
}
|
||||||
|
|
||||||
public void testBuiltRemoteClustersSeeds() throws Exception {
|
public void testBuiltRemoteClustersSeeds() throws Exception {
|
||||||
Map<String, List<DiscoveryNode>> map = RemoteClusterService.buildRemoteClustersSeeds(
|
Map<String, List<Supplier<DiscoveryNode>>> map = RemoteClusterService.buildRemoteClustersSeeds(
|
||||||
Settings.builder().put("search.remote.foo.seeds", "192.168.0.1:8080").put("search.remote.bar.seeds", "[::1]:9090").build());
|
Settings.builder().put("search.remote.foo.seeds", "192.168.0.1:8080").put("search.remote.bar.seeds", "[::1]:9090").build());
|
||||||
assertEquals(2, map.size());
|
assertEquals(2, map.size());
|
||||||
assertTrue(map.containsKey("foo"));
|
assertTrue(map.containsKey("foo"));
|
||||||
|
@ -114,13 +124,13 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
assertEquals(1, map.get("foo").size());
|
assertEquals(1, map.get("foo").size());
|
||||||
assertEquals(1, map.get("bar").size());
|
assertEquals(1, map.get("bar").size());
|
||||||
|
|
||||||
DiscoveryNode foo = map.get("foo").get(0);
|
DiscoveryNode foo = map.get("foo").get(0).get();
|
||||||
|
|
||||||
assertEquals(foo.getAddress(), new TransportAddress(new InetSocketAddress(InetAddress.getByName("192.168.0.1"), 8080)));
|
assertEquals(foo.getAddress(), new TransportAddress(new InetSocketAddress(InetAddress.getByName("192.168.0.1"), 8080)));
|
||||||
assertEquals(foo.getId(), "foo#192.168.0.1:8080");
|
assertEquals(foo.getId(), "foo#192.168.0.1:8080");
|
||||||
assertEquals(foo.getVersion(), Version.CURRENT.minimumCompatibilityVersion());
|
assertEquals(foo.getVersion(), Version.CURRENT.minimumCompatibilityVersion());
|
||||||
|
|
||||||
DiscoveryNode bar = map.get("bar").get(0);
|
DiscoveryNode bar = map.get("bar").get(0).get();
|
||||||
assertEquals(bar.getAddress(), new TransportAddress(new InetSocketAddress(InetAddress.getByName("[::1]"), 9090)));
|
assertEquals(bar.getAddress(), new TransportAddress(new InetSocketAddress(InetAddress.getByName("[::1]"), 9090)));
|
||||||
assertEquals(bar.getId(), "bar#[::1]:9090");
|
assertEquals(bar.getId(), "bar#[::1]:9090");
|
||||||
assertEquals(bar.getVersion(), Version.CURRENT.minimumCompatibilityVersion());
|
assertEquals(bar.getVersion(), Version.CURRENT.minimumCompatibilityVersion());
|
||||||
|
@ -194,10 +204,10 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
assertFalse(service.isCrossClusterSearchEnabled());
|
assertFalse(service.isCrossClusterSearchEnabled());
|
||||||
service.initializeRemoteClusters();
|
service.initializeRemoteClusters();
|
||||||
assertFalse(service.isCrossClusterSearchEnabled());
|
assertFalse(service.isCrossClusterSearchEnabled());
|
||||||
service.updateRemoteCluster("cluster_1", Collections.singletonList(seedNode.getAddress().address()));
|
service.updateRemoteCluster("cluster_1", Collections.singletonList(seedNode.getAddress().toString()));
|
||||||
assertTrue(service.isCrossClusterSearchEnabled());
|
assertTrue(service.isCrossClusterSearchEnabled());
|
||||||
assertTrue(service.isRemoteClusterRegistered("cluster_1"));
|
assertTrue(service.isRemoteClusterRegistered("cluster_1"));
|
||||||
service.updateRemoteCluster("cluster_2", Collections.singletonList(otherSeedNode.getAddress().address()));
|
service.updateRemoteCluster("cluster_2", Collections.singletonList(otherSeedNode.getAddress().toString()));
|
||||||
assertTrue(service.isCrossClusterSearchEnabled());
|
assertTrue(service.isCrossClusterSearchEnabled());
|
||||||
assertTrue(service.isRemoteClusterRegistered("cluster_1"));
|
assertTrue(service.isRemoteClusterRegistered("cluster_1"));
|
||||||
assertTrue(service.isRemoteClusterRegistered("cluster_2"));
|
assertTrue(service.isRemoteClusterRegistered("cluster_2"));
|
||||||
|
@ -252,22 +262,17 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
service.initializeRemoteClusters();
|
service.initializeRemoteClusters();
|
||||||
assertFalse(service.isCrossClusterSearchEnabled());
|
assertFalse(service.isCrossClusterSearchEnabled());
|
||||||
|
|
||||||
final InetSocketAddress c1N1Address = c1N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c1N2Address = c1N2Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N1Address = c2N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N2Address = c2N2Node.getAddress().address();
|
|
||||||
|
|
||||||
final CountDownLatch firstLatch = new CountDownLatch(1);
|
final CountDownLatch firstLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_1",
|
"cluster_1",
|
||||||
Arrays.asList(c1N1Address, c1N2Address),
|
Arrays.asList(c1N1Node.getAddress().toString(), c1N2Node.getAddress().toString()),
|
||||||
connectionListener(firstLatch));
|
connectionListener(firstLatch));
|
||||||
firstLatch.await();
|
firstLatch.await();
|
||||||
|
|
||||||
final CountDownLatch secondLatch = new CountDownLatch(1);
|
final CountDownLatch secondLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_2",
|
"cluster_2",
|
||||||
Arrays.asList(c2N1Address, c2N2Address),
|
Arrays.asList(c2N1Node.getAddress().toString(), c2N2Node.getAddress().toString()),
|
||||||
connectionListener(secondLatch));
|
connectionListener(secondLatch));
|
||||||
secondLatch.await();
|
secondLatch.await();
|
||||||
|
|
||||||
|
@ -321,22 +326,17 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
service.initializeRemoteClusters();
|
service.initializeRemoteClusters();
|
||||||
assertFalse(service.isCrossClusterSearchEnabled());
|
assertFalse(service.isCrossClusterSearchEnabled());
|
||||||
|
|
||||||
final InetSocketAddress c1N1Address = c1N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c1N2Address = c1N2Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N1Address = c2N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N2Address = c2N2Node.getAddress().address();
|
|
||||||
|
|
||||||
final CountDownLatch firstLatch = new CountDownLatch(1);
|
final CountDownLatch firstLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_1",
|
"cluster_1",
|
||||||
Arrays.asList(c1N1Address, c1N2Address),
|
Arrays.asList(c1N1Node.getAddress().toString(), c1N2Node.getAddress().toString()),
|
||||||
connectionListener(firstLatch));
|
connectionListener(firstLatch));
|
||||||
firstLatch.await();
|
firstLatch.await();
|
||||||
|
|
||||||
final CountDownLatch secondLatch = new CountDownLatch(1);
|
final CountDownLatch secondLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_2",
|
"cluster_2",
|
||||||
Arrays.asList(c2N1Address, c2N2Address),
|
Arrays.asList(c2N1Node.getAddress().toString(), c2N2Node.getAddress().toString()),
|
||||||
connectionListener(secondLatch));
|
connectionListener(secondLatch));
|
||||||
secondLatch.await();
|
secondLatch.await();
|
||||||
|
|
||||||
|
@ -398,22 +398,17 @@ public class RemoteClusterServiceTests extends ESTestCase {
|
||||||
service.initializeRemoteClusters();
|
service.initializeRemoteClusters();
|
||||||
assertFalse(service.isCrossClusterSearchEnabled());
|
assertFalse(service.isCrossClusterSearchEnabled());
|
||||||
|
|
||||||
final InetSocketAddress c1N1Address = c1N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c1N2Address = c1N2Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N1Address = c2N1Node.getAddress().address();
|
|
||||||
final InetSocketAddress c2N2Address = c2N2Node.getAddress().address();
|
|
||||||
|
|
||||||
final CountDownLatch firstLatch = new CountDownLatch(1);
|
final CountDownLatch firstLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_1",
|
"cluster_1",
|
||||||
Arrays.asList(c1N1Address, c1N2Address),
|
Arrays.asList(c1N1Node.getAddress().toString(), c1N2Node.getAddress().toString()),
|
||||||
connectionListener(firstLatch));
|
connectionListener(firstLatch));
|
||||||
firstLatch.await();
|
firstLatch.await();
|
||||||
|
|
||||||
final CountDownLatch secondLatch = new CountDownLatch(1);
|
final CountDownLatch secondLatch = new CountDownLatch(1);
|
||||||
service.updateRemoteCluster(
|
service.updateRemoteCluster(
|
||||||
"cluster_2",
|
"cluster_2",
|
||||||
Arrays.asList(c2N1Address, c2N2Address),
|
Arrays.asList(c2N1Node.getAddress().toString(), c2N2Node.getAddress().toString()),
|
||||||
connectionListener(secondLatch));
|
connectionListener(secondLatch));
|
||||||
secondLatch.await();
|
secondLatch.await();
|
||||||
CountDownLatch latch = new CountDownLatch(1);
|
CountDownLatch latch = new CountDownLatch(1);
|
||||||
|
|
|
@ -30,7 +30,6 @@ import org.elasticsearch.transport.TransportRequest;
|
||||||
import org.elasticsearch.xpack.core.graph.action.GraphExploreRequest;
|
import org.elasticsearch.xpack.core.graph.action.GraphExploreRequest;
|
||||||
import org.elasticsearch.xpack.core.security.authz.IndicesAndAliasesResolverField;
|
import org.elasticsearch.xpack.core.security.authz.IndicesAndAliasesResolverField;
|
||||||
|
|
||||||
import java.net.InetSocketAddress;
|
|
||||||
import java.util.ArrayList;
|
import java.util.ArrayList;
|
||||||
import java.util.Arrays;
|
import java.util.Arrays;
|
||||||
import java.util.Collections;
|
import java.util.Collections;
|
||||||
|
@ -429,7 +428,7 @@ class IndicesAndAliasesResolver {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void updateRemoteCluster(String clusterAlias, List<InetSocketAddress> addresses) {
|
protected void updateRemoteCluster(String clusterAlias, List<String> addresses) {
|
||||||
if (addresses.isEmpty()) {
|
if (addresses.isEmpty()) {
|
||||||
clusters.remove(clusterAlias);
|
clusters.remove(clusterAlias);
|
||||||
} else {
|
} else {
|
||||||
|
|
Loading…
Reference in New Issue