mirror of https://github.com/apache/druid.git
Make GuavaUtilsTest use less CPU (#14487)
This commit is contained in:
parent
6ba10c8b6c
commit
2f0a43790c
|
@ -31,9 +31,11 @@ import org.junit.Test;
|
||||||
import java.util.ArrayList;
|
import java.util.ArrayList;
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
import java.util.concurrent.Callable;
|
import java.util.concurrent.Callable;
|
||||||
|
import java.util.concurrent.CountDownLatch;
|
||||||
|
import java.util.concurrent.ExecutionException;
|
||||||
import java.util.concurrent.ExecutorService;
|
import java.util.concurrent.ExecutorService;
|
||||||
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.concurrent.atomic.AtomicBoolean;
|
import java.util.concurrent.atomic.AtomicBoolean;
|
||||||
import java.util.concurrent.atomic.AtomicInteger;
|
|
||||||
import java.util.function.Function;
|
import java.util.function.Function;
|
||||||
|
|
||||||
public class GuavaUtilsTest
|
public class GuavaUtilsTest
|
||||||
|
@ -73,23 +75,23 @@ public class GuavaUtilsTest
|
||||||
int tasks = 3;
|
int tasks = 3;
|
||||||
ExecutorService service = Execs.multiThreaded(tasks, "GuavaUtilsTest-%d");
|
ExecutorService service = Execs.multiThreaded(tasks, "GuavaUtilsTest-%d");
|
||||||
ListeningExecutorService exc = MoreExecutors.listeningDecorator(service);
|
ListeningExecutorService exc = MoreExecutors.listeningDecorator(service);
|
||||||
AtomicInteger index = new AtomicInteger(0);
|
|
||||||
//a flag what time to throw exception.
|
//a flag what time to throw exception.
|
||||||
AtomicBoolean active = new AtomicBoolean(false);
|
AtomicBoolean someoneFailed = new AtomicBoolean(false);
|
||||||
|
List<CountDownLatch> latches = new ArrayList<>(tasks);
|
||||||
Function<Integer, List<ListenableFuture<Object>>> function = (taskCount) -> {
|
Function<Integer, List<ListenableFuture<Object>>> function = (taskCount) -> {
|
||||||
List<ListenableFuture<Object>> futures = new ArrayList<>();
|
List<ListenableFuture<Object>> futures = new ArrayList<>();
|
||||||
for (int i = 0; i < taskCount; i++) {
|
for (int i = 0; i < taskCount; i++) {
|
||||||
|
final CountDownLatch latch = new CountDownLatch(1);
|
||||||
|
latches.add(latch);
|
||||||
ListenableFuture<Object> future = exc.submit(new Callable<Object>() {
|
ListenableFuture<Object> future = exc.submit(new Callable<Object>() {
|
||||||
@Override
|
@Override
|
||||||
public Object call() throws RuntimeException
|
public Object call() throws RuntimeException, InterruptedException
|
||||||
{
|
{
|
||||||
int internalIndex = index.incrementAndGet();
|
latch.await(60, TimeUnit.SECONDS);
|
||||||
while (true) {
|
if (someoneFailed.compareAndSet(false, true)) {
|
||||||
if (internalIndex == taskCount && active.get()) {
|
throw new RuntimeException("This exception simulates an error");
|
||||||
//here we simulate occurs exception in some one future.
|
|
||||||
throw new RuntimeException("A big bug");
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
return null;
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
futures.add(future);
|
futures.add(future);
|
||||||
|
@ -99,17 +101,20 @@ public class GuavaUtilsTest
|
||||||
|
|
||||||
List<ListenableFuture<Object>> futures = function.apply(tasks);
|
List<ListenableFuture<Object>> futures = function.apply(tasks);
|
||||||
Assert.assertEquals(tasks, futures.stream().filter(f -> !f.isDone()).count());
|
Assert.assertEquals(tasks, futures.stream().filter(f -> !f.isDone()).count());
|
||||||
//here we make one of task throw exception.
|
// "release" the last tasks, which will cause it to fail as someoneFailed will still be false
|
||||||
active.set(true);
|
latches.get(tasks - 1).countDown();
|
||||||
|
|
||||||
ListenableFuture<List<Object>> future = Futures.allAsList(futures);
|
ListenableFuture<List<Object>> future = Futures.allAsList(futures);
|
||||||
try {
|
|
||||||
future.get();
|
ExecutionException thrown = Assert.assertThrows(
|
||||||
}
|
ExecutionException.class,
|
||||||
catch (Exception e) {
|
future::get
|
||||||
Assert.assertEquals("java.lang.RuntimeException: A big bug", e.getMessage());
|
);
|
||||||
|
Assert.assertEquals("This exception simulates an error", thrown.getCause().getMessage());
|
||||||
GuavaUtils.cancelAll(true, future, futures);
|
GuavaUtils.cancelAll(true, future, futures);
|
||||||
Assert.assertEquals(0, futures.stream().filter(f -> !f.isDone()).count());
|
Assert.assertEquals(0, futures.stream().filter(f -> !f.isDone()).count());
|
||||||
|
for (CountDownLatch latch : latches) {
|
||||||
|
latch.countDown();
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
Loading…
Reference in New Issue