mirror of
https://github.com/honeymoose/OpenSearch.git
synced 2025-02-25 06:16:40 +00:00
UnicastZenPingTests didn't properly wait for pinging rounds to be closed
The test ping and waited for the ping results to be returned but since we first return the result and then close temporary connections, assertions are tripped that expects all connections to close by end of test . Closes #22497
This commit is contained in:
parent
cb8f8fc9ad
commit
be0c461b50
@ -72,7 +72,6 @@ import java.util.List;
|
|||||||
import java.util.Map;
|
import java.util.Map;
|
||||||
import java.util.Set;
|
import java.util.Set;
|
||||||
import java.util.Stack;
|
import java.util.Stack;
|
||||||
import java.util.concurrent.CompletableFuture;
|
|
||||||
import java.util.concurrent.ConcurrentMap;
|
import java.util.concurrent.ConcurrentMap;
|
||||||
import java.util.concurrent.CountDownLatch;
|
import java.util.concurrent.CountDownLatch;
|
||||||
import java.util.concurrent.ExecutionException;
|
import java.util.concurrent.ExecutionException;
|
||||||
@ -80,6 +79,7 @@ import java.util.concurrent.ExecutorService;
|
|||||||
import java.util.concurrent.ThreadFactory;
|
import java.util.concurrent.ThreadFactory;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.concurrent.atomic.AtomicInteger;
|
import java.util.concurrent.atomic.AtomicInteger;
|
||||||
|
import java.util.concurrent.atomic.AtomicReference;
|
||||||
import java.util.function.BiFunction;
|
import java.util.function.BiFunction;
|
||||||
import java.util.stream.Collectors;
|
import java.util.stream.Collectors;
|
||||||
import java.util.stream.IntStream;
|
import java.util.stream.IntStream;
|
||||||
@ -797,21 +797,22 @@ public class UnicastZenPingTests extends ESTestCase {
|
|||||||
|
|
||||||
volatile CountDownLatch allTasksCompleted;
|
volatile CountDownLatch allTasksCompleted;
|
||||||
volatile AtomicInteger pendingTasks;
|
volatile AtomicInteger pendingTasks;
|
||||||
|
volatile CountDownLatch pingingRoundClosed;
|
||||||
|
|
||||||
PingCollection pingAndWait() throws ExecutionException, InterruptedException {
|
PingCollection pingAndWait() throws ExecutionException, InterruptedException {
|
||||||
allTasksCompleted = new CountDownLatch(1);
|
allTasksCompleted = new CountDownLatch(1);
|
||||||
|
pingingRoundClosed = new CountDownLatch(1);
|
||||||
pendingTasks = new AtomicInteger();
|
pendingTasks = new AtomicInteger();
|
||||||
// make the three sending rounds to come as started
|
// mark the three sending rounds as ongoing
|
||||||
markTaskAsStarted("send pings");
|
markTaskAsStarted("send pings");
|
||||||
markTaskAsStarted("send pings");
|
markTaskAsStarted("send pings");
|
||||||
markTaskAsStarted("send pings");
|
markTaskAsStarted("send pings");
|
||||||
final CompletableFuture<PingCollection> response = new CompletableFuture<>();
|
final AtomicReference<PingCollection> response = new AtomicReference<>();
|
||||||
try {
|
ping(response::set, TimeValue.timeValueMillis(1), TimeValue.timeValueSeconds(1));
|
||||||
ping(response::complete, TimeValue.timeValueMillis(1), TimeValue.timeValueSeconds(1));
|
pingingRoundClosed.await();
|
||||||
} catch (Exception ex) {
|
final PingCollection result = response.get();
|
||||||
response.completeExceptionally(ex);
|
assertNotNull("pinging didn't complete", result);
|
||||||
}
|
return result;
|
||||||
return response.get();
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
@ -823,6 +824,7 @@ public class UnicastZenPingTests extends ESTestCase {
|
|||||||
// ok, finish anyway
|
// ok, finish anyway
|
||||||
}
|
}
|
||||||
super.finishPingingRound(pingingRound);
|
super.finishPingingRound(pingingRound);
|
||||||
|
pingingRoundClosed.countDown();
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
Loading…
x
Reference in New Issue
Block a user