YARN-5857. TestLogAggregationService.testFixedSizeThreadPool fails intermittently on trunk. Contributed by Bilwa S T.
This commit is contained in:
parent
10ec31d20e
commit
e684b17e6f
|
@ -58,6 +58,7 @@ import java.util.Map;
|
||||||
import java.util.Set;
|
import java.util.Set;
|
||||||
import java.util.concurrent.ConcurrentHashMap;
|
import java.util.concurrent.ConcurrentHashMap;
|
||||||
import java.util.concurrent.ConcurrentMap;
|
import java.util.concurrent.ConcurrentMap;
|
||||||
|
import java.util.concurrent.CountDownLatch;
|
||||||
import java.util.concurrent.ExecutorService;
|
import java.util.concurrent.ExecutorService;
|
||||||
import java.util.concurrent.ThreadPoolExecutor;
|
import java.util.concurrent.ThreadPoolExecutor;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
|
@ -1215,6 +1216,8 @@ public class TestLogAggregationService extends BaseContainerManagerTest {
|
||||||
|
|
||||||
ExecutorService executorService = logAggregationService.threadPool;
|
ExecutorService executorService = logAggregationService.threadPool;
|
||||||
|
|
||||||
|
CountDownLatch latch = new CountDownLatch(threadPoolSize);
|
||||||
|
|
||||||
// used to block threads in the thread pool because main thread always
|
// used to block threads in the thread pool because main thread always
|
||||||
// acquires the write lock first.
|
// acquires the write lock first.
|
||||||
final ReadWriteLock rwLock = new ReentrantReadWriteLock();
|
final ReadWriteLock rwLock = new ReentrantReadWriteLock();
|
||||||
|
@ -1227,8 +1230,9 @@ public class TestLogAggregationService extends BaseContainerManagerTest {
|
||||||
@Override
|
@Override
|
||||||
public void run() {
|
public void run() {
|
||||||
try {
|
try {
|
||||||
|
latch.countDown();
|
||||||
// threads in the thread pool running this will be blocked
|
// threads in the thread pool running this will be blocked
|
||||||
rLock.tryLock(35000, TimeUnit.MILLISECONDS);
|
rLock.tryLock(15000, TimeUnit.MILLISECONDS);
|
||||||
} catch (InterruptedException e) {
|
} catch (InterruptedException e) {
|
||||||
e.printStackTrace();
|
e.printStackTrace();
|
||||||
} finally {
|
} finally {
|
||||||
|
@ -1244,6 +1248,7 @@ public class TestLogAggregationService extends BaseContainerManagerTest {
|
||||||
executorService.submit(runnable);
|
executorService.submit(runnable);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
latch.await();
|
||||||
// count the number of current running LogAggregationService threads
|
// count the number of current running LogAggregationService threads
|
||||||
int runningThread = ((ThreadPoolExecutor) executorService).getActiveCount();
|
int runningThread = ((ThreadPoolExecutor) executorService).getActiveCount();
|
||||||
assertEquals(threadPoolSize, runningThread);
|
assertEquals(threadPoolSize, runningThread);
|
||||||
|
|
Loading…
Reference in New Issue