Skip to content

Commit e2fc5a2

Browse files
committed
Simplify handling of hostname lookups in UZP
This commit simplifies the handling of hostname lookups in UnicastZenPing, removing an unnecessary abstraction and, subsequently, an unnecessary method.
1 parent a28dc6e commit e2fc5a2

File tree

1 file changed

+20
-76
lines changed

1 file changed

+20
-76
lines changed

core/src/main/java/org/elasticsearch/discovery/zen/UnicastZenPing.java

Lines changed: 20 additions & 76 deletions
Original file line numberDiff line numberDiff line change
@@ -58,7 +58,6 @@
5858
import org.elasticsearch.transport.TransportService;
5959

6060
import java.io.IOException;
61-
import java.net.UnknownHostException;
6261
import java.util.ArrayList;
6362
import java.util.Arrays;
6463
import java.util.Collection;
@@ -176,40 +175,6 @@ public UnicastZenPing(Settings settings, ThreadPool threadPool, TransportService
176175
threadPool.getThreadContext());
177176
}
178177

179-
private static class ResolvedHostname {
180-
181-
private final TransportAddress[] addresses;
182-
private final UnknownHostException failure;
183-
184-
public static ResolvedHostname success(final TransportAddress[] addresses) {
185-
return new ResolvedHostname(addresses, null);
186-
}
187-
188-
public static ResolvedHostname failure(final UnknownHostException failure) {
189-
return new ResolvedHostname(null, failure);
190-
}
191-
192-
private ResolvedHostname(final TransportAddress[] addresses, UnknownHostException failure) {
193-
assert addresses != null && failure == null || addresses == null && failure != null;
194-
this.addresses = addresses;
195-
this.failure = failure;
196-
}
197-
198-
public boolean isSuccess() {
199-
return addresses != null;
200-
}
201-
202-
public TransportAddress[] addresses() {
203-
return addresses;
204-
}
205-
206-
public UnknownHostException failure() {
207-
assert !isSuccess();
208-
return failure;
209-
}
210-
211-
}
212-
213178
/**
214179
* Resolves a list of hosts to a list of discovery nodes. Each host is resolved into a transport address (or a collection of addresses
215180
* if the number of ports is greater than one) and the transport addresses are used to created discovery nodes. Host lookups are done
@@ -242,37 +207,37 @@ public static List<DiscoveryNode> resolveDiscoveryNodes(
242207
throw new IllegalArgumentException("resolve timeout must be non-negative but was [" + resolveTimeout + "]");
243208
}
244209
// create tasks to submit to the executor service; we will wait up to resolveTimeout for these tasks to complete
245-
final List<Callable<ResolvedHostname>> callables =
246-
hosts.stream().map(hn -> lookup(hn, transportService, limitPortCounts)).collect(Collectors.toList());
247-
final List<Future<ResolvedHostname>> futures =
210+
final List<Callable<TransportAddress[]>> callables =
211+
hosts
212+
.stream()
213+
.map(hn -> (Callable<TransportAddress[]>)() -> transportService.addressesFromString(hn, limitPortCounts))
214+
.collect(Collectors.toList());
215+
final List<Future<TransportAddress[]>> futures =
248216
executorService.invokeAll(callables, resolveTimeout.nanos(), TimeUnit.NANOSECONDS);
249217
final List<DiscoveryNode> discoveryNodes = new ArrayList<>();
250218
// ExecutorService#invokeAll guarantees that the futures are returned in the iteration order of the tasks so we can associate the
251219
// hostname with the corresponding task by iterating together
252220
final Iterator<String> it = hosts.iterator();
253-
for (final Future<ResolvedHostname> future : futures) {
221+
for (final Future<TransportAddress[]> future : futures) {
254222
final String hostname = it.next();
255223
if (!future.isCancelled()) {
224+
assert future.isDone();
256225
try {
257-
final ResolvedHostname resolvedHostname = future.get();
258-
if (resolvedHostname.isSuccess()) {
259-
logger.trace("resolved host [{}] to {}", hostname, resolvedHostname.addresses());
260-
for (final TransportAddress address : resolvedHostname.addresses()) {
261-
discoveryNodes.add(
262-
new DiscoveryNode(
263-
idGenerator.get(),
264-
address,
265-
emptyMap(),
266-
emptySet(),
267-
Version.CURRENT.minimumCompatibilityVersion()));
268-
}
269-
} else {
270-
final String message = "failed to resolve host [" + hostname + "]";
271-
logger.warn(message, resolvedHostname.failure());
226+
final TransportAddress[] addresses = future.get();
227+
logger.trace("resolved host [{}] to {}", hostname, addresses);
228+
for (final TransportAddress address : addresses) {
229+
discoveryNodes.add(
230+
new DiscoveryNode(
231+
idGenerator.get(),
232+
address,
233+
emptyMap(),
234+
emptySet(),
235+
Version.CURRENT.minimumCompatibilityVersion()));
272236
}
273237
} catch (final ExecutionException e) {
238+
assert e.getCause() != null;
274239
final String message = "failed to resolve host [" + hostname + "]";
275-
logger.warn(message, e);
240+
logger.warn(message, e.getCause());
276241
}
277242
} else {
278243
logger.warn("timed out after [{}] resolving host [{}]", resolveTimeout, hostname);
@@ -281,27 +246,6 @@ public static List<DiscoveryNode> resolveDiscoveryNodes(
281246
return discoveryNodes;
282247
}
283248

284-
/**
285-
* Creates a callable for looking up the specified host.
286-
*
287-
* @param host the host to lookup
288-
* @param transportService the transport service to use for lookups
289-
* @param limitPortCounts the port count limit
290-
* @return a callable that can be used to submit to an executor service
291-
*/
292-
private static Callable<ResolvedHostname> lookup(
293-
final String host,
294-
final TransportService transportService,
295-
final int limitPortCounts) {
296-
return () -> {
297-
try {
298-
return ResolvedHostname.success(transportService.addressesFromString(host, limitPortCounts));
299-
} catch (final UnknownHostException e) {
300-
return ResolvedHostname.failure(e);
301-
}
302-
};
303-
}
304-
305249
@Override
306250
public void close() {
307251
ThreadPool.terminate(unicastZenPingExecutorService, 0, TimeUnit.SECONDS);

0 commit comments

Comments
 (0)