|
@@ -55,6 +55,12 @@ 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.ExecutorService;
|
|
|
|
+import java.util.concurrent.ThreadPoolExecutor;
|
|
|
|
+import java.util.concurrent.TimeUnit;
|
|
|
|
+import java.util.concurrent.locks.Lock;
|
|
|
|
+import java.util.concurrent.locks.ReadWriteLock;
|
|
|
|
+import java.util.concurrent.locks.ReentrantReadWriteLock;
|
|
|
|
|
|
import org.apache.commons.lang.StringUtils;
|
|
import org.apache.commons.lang.StringUtils;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.LogFactory;
|
|
@@ -1040,6 +1046,143 @@ public class TestLogAggregationService extends BaseContainerManagerTest {
|
|
return appAcls;
|
|
return appAcls;
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ @Test (timeout = 30000)
|
|
|
|
+ public void testFixedSizeThreadPool() throws Exception {
|
|
|
|
+ // store configured thread pool size temporarily for restoration
|
|
|
|
+ int initThreadPoolSize = conf.getInt(YarnConfiguration
|
|
|
|
+ .NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ YarnConfiguration.DEFAULT_NM_LOG_AGGREGATION_THREAD_POOL_SIZE);
|
|
|
|
+
|
|
|
|
+ int threadPoolSize = 3;
|
|
|
|
+ conf.setInt(YarnConfiguration.NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ threadPoolSize);
|
|
|
|
+
|
|
|
|
+ DeletionService delSrvc = mock(DeletionService.class);
|
|
|
|
+
|
|
|
|
+ LocalDirsHandlerService dirSvc = mock(LocalDirsHandlerService.class);
|
|
|
|
+ when(dirSvc.getLogDirs()).thenThrow(new RuntimeException());
|
|
|
|
+
|
|
|
|
+ LogAggregationService logAggregationService =
|
|
|
|
+ new LogAggregationService(dispatcher, this.context, delSrvc, dirSvc);
|
|
|
|
+
|
|
|
|
+ logAggregationService.init(this.conf);
|
|
|
|
+ logAggregationService.start();
|
|
|
|
+
|
|
|
|
+ ExecutorService executorService = logAggregationService.threadPool;
|
|
|
|
+
|
|
|
|
+ // used to block threads in the thread pool because main thread always
|
|
|
|
+ // acquires the write lock first.
|
|
|
|
+ final ReadWriteLock rwLock = new ReentrantReadWriteLock();
|
|
|
|
+ final Lock rLock = rwLock.readLock();
|
|
|
|
+ final Lock wLock = rwLock.writeLock();
|
|
|
|
+
|
|
|
|
+ try {
|
|
|
|
+ wLock.lock();
|
|
|
|
+ Runnable runnable = new Runnable() {
|
|
|
|
+ @Override
|
|
|
|
+ public void run() {
|
|
|
|
+ try {
|
|
|
|
+ // threads in the thread pool running this will be blocked
|
|
|
|
+ rLock.tryLock(35000, TimeUnit.MILLISECONDS);
|
|
|
|
+ } catch (InterruptedException e) {
|
|
|
|
+ e.printStackTrace();
|
|
|
|
+ } finally {
|
|
|
|
+ rLock.unlock();
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ };
|
|
|
|
+
|
|
|
|
+ // submit $(threadPoolSize + 1) runnables to the thread pool. If the thread
|
|
|
|
+ // pool size is set properly, only $(threadPoolSize) threads will be
|
|
|
|
+ // created in the thread pool, each of which is blocked on the read lock.
|
|
|
|
+ for(int i = 0; i < threadPoolSize + 1; i++) {
|
|
|
|
+ executorService.submit(runnable);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ // count the number of current running LogAggregationService threads
|
|
|
|
+ int runningThread = ((ThreadPoolExecutor) executorService).getActiveCount();
|
|
|
|
+ assertEquals(threadPoolSize, runningThread);
|
|
|
|
+ }
|
|
|
|
+ finally {
|
|
|
|
+ wLock.unlock();
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ logAggregationService.stop();
|
|
|
|
+ logAggregationService.close();
|
|
|
|
+
|
|
|
|
+ // restore the original configurations to avoid side effects
|
|
|
|
+ conf.setInt(YarnConfiguration.NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ initThreadPoolSize);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testInvalidThreadPoolSizeNaN() throws IOException {
|
|
|
|
+ testInvalidThreadPoolSizeValue("NaN");
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testInvalidThreadPoolSizeNegative() throws IOException {
|
|
|
|
+ testInvalidThreadPoolSizeValue("-100");
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testInvalidThreadPoolSizeXLarge() throws IOException {
|
|
|
|
+ testInvalidThreadPoolSizeValue("11111111111");
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private void testInvalidThreadPoolSizeValue(final String threadPoolSize)
|
|
|
|
+ throws IOException {
|
|
|
|
+ Supplier<Boolean> isInputInvalid = new Supplier<Boolean>() {
|
|
|
|
+ @Override
|
|
|
|
+ public Boolean get() {
|
|
|
|
+ try {
|
|
|
|
+ int value = Integer.parseInt(threadPoolSize);
|
|
|
|
+ return value <= 0;
|
|
|
|
+ } catch (NumberFormatException ex) {
|
|
|
|
+ return true;
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ };
|
|
|
|
+
|
|
|
|
+ assertTrue("The thread pool size must be invalid to use with this " +
|
|
|
|
+ "method", isInputInvalid.get());
|
|
|
|
+
|
|
|
|
+
|
|
|
|
+ // store configured thread pool size temporarily for restoration
|
|
|
|
+ int initThreadPoolSize = conf.getInt(YarnConfiguration
|
|
|
|
+ .NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ YarnConfiguration.DEFAULT_NM_LOG_AGGREGATION_THREAD_POOL_SIZE);
|
|
|
|
+
|
|
|
|
+ conf.set(YarnConfiguration.NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ threadPoolSize);
|
|
|
|
+
|
|
|
|
+ DeletionService delSrvc = mock(DeletionService.class);
|
|
|
|
+
|
|
|
|
+ LocalDirsHandlerService dirSvc = mock(LocalDirsHandlerService.class);
|
|
|
|
+ when(dirSvc.getLogDirs()).thenThrow(new RuntimeException());
|
|
|
|
+
|
|
|
|
+ LogAggregationService logAggregationService =
|
|
|
|
+ new LogAggregationService(dispatcher, this.context, delSrvc, dirSvc);
|
|
|
|
+
|
|
|
|
+ logAggregationService.init(this.conf);
|
|
|
|
+ logAggregationService.start();
|
|
|
|
+
|
|
|
|
+ ThreadPoolExecutor executorService = (ThreadPoolExecutor)
|
|
|
|
+ logAggregationService.threadPool;
|
|
|
|
+ assertEquals("The thread pool size should be set to the value of YARN" +
|
|
|
|
+ ".DEFAULT_NM_LOG_AGGREGATION_THREAD_POOL_SIZE because the configured "
|
|
|
|
+ + " thread pool size is " + "invalid.",
|
|
|
|
+ YarnConfiguration.DEFAULT_NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ executorService.getMaximumPoolSize());
|
|
|
|
+
|
|
|
|
+ logAggregationService.stop();
|
|
|
|
+ logAggregationService.close();
|
|
|
|
+
|
|
|
|
+ // retore original configuration to aviod side effects
|
|
|
|
+ conf.setInt(YarnConfiguration.NM_LOG_AGGREGATION_THREAD_POOL_SIZE,
|
|
|
|
+ initThreadPoolSize);
|
|
|
|
+ }
|
|
|
|
+
|
|
@Test(timeout=20000)
|
|
@Test(timeout=20000)
|
|
public void testStopAfterError() throws Exception {
|
|
public void testStopAfterError() throws Exception {
|
|
DeletionService delSrvc = mock(DeletionService.class);
|
|
DeletionService delSrvc = mock(DeletionService.class);
|