mirror of https://github.com/apache/druid.git
review comments
This commit is contained in:
parent
fb819abd6f
commit
3db33c3268
|
@ -60,7 +60,7 @@ public class Execs
|
||||||
* @param capacity maximum capacity after which the executorService will block on accepting new tasks
|
* @param capacity maximum capacity after which the executorService will block on accepting new tasks
|
||||||
* @return ExecutorService which blocks accepting new tasks when the capacity reached
|
* @return ExecutorService which blocks accepting new tasks when the capacity reached
|
||||||
*/
|
*/
|
||||||
public static ExecutorService blockingSingleThreaded(String nameFormat, int capacity)
|
public static ExecutorService newBlockingSingleThreaded(String nameFormat, int capacity)
|
||||||
{
|
{
|
||||||
return new ThreadPoolExecutor(
|
return new ThreadPoolExecutor(
|
||||||
1, 1,
|
1, 1,
|
||||||
|
|
|
@ -1,3 +1,22 @@
|
||||||
|
/*
|
||||||
|
* Druid - a distributed column store.
|
||||||
|
* Copyright (C) 2012, 2013, 2014 Metamarkets Group Inc.
|
||||||
|
*
|
||||||
|
* This program is free software; you can redistribute it and/or
|
||||||
|
* modify it under the terms of the GNU General Public License
|
||||||
|
* as published by the Free Software Foundation; either version 2
|
||||||
|
* of the License, or (at your option) any later version.
|
||||||
|
*
|
||||||
|
* This program is distributed in the hope that it will be useful,
|
||||||
|
* but WITHOUT ANY WARRANTY; without even the implied warranty of
|
||||||
|
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
|
||||||
|
* GNU General Public License for more details.
|
||||||
|
*
|
||||||
|
* You should have received a copy of the GNU General Public License
|
||||||
|
* along with this program; if not, write to the Free Software
|
||||||
|
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
|
||||||
|
*/
|
||||||
|
|
||||||
package io.druid.concurrent;
|
package io.druid.concurrent;
|
||||||
|
|
||||||
import com.google.common.base.Throwables;
|
import com.google.common.base.Throwables;
|
||||||
|
@ -5,7 +24,9 @@ import org.junit.Assert;
|
||||||
import org.junit.Test;
|
import org.junit.Test;
|
||||||
|
|
||||||
import java.util.concurrent.CyclicBarrier;
|
import java.util.concurrent.CyclicBarrier;
|
||||||
|
import java.util.concurrent.Executor;
|
||||||
import java.util.concurrent.ExecutorService;
|
import java.util.concurrent.ExecutorService;
|
||||||
|
import java.util.concurrent.Executors;
|
||||||
import java.util.concurrent.atomic.AtomicInteger;
|
import java.util.concurrent.atomic.AtomicInteger;
|
||||||
|
|
||||||
public class ExecsTest
|
public class ExecsTest
|
||||||
|
@ -14,50 +35,59 @@ public class ExecsTest
|
||||||
public void testBlockingExecutorService() throws Exception
|
public void testBlockingExecutorService() throws Exception
|
||||||
{
|
{
|
||||||
final int capacity = 3;
|
final int capacity = 3;
|
||||||
final ExecutorService executorService = Execs.blockingSingleThreaded("test%d", capacity);
|
final ExecutorService executorService = Execs.newBlockingSingleThreaded("test%d", capacity);
|
||||||
final AtomicInteger producedCount = new AtomicInteger();
|
final AtomicInteger producedCount = new AtomicInteger();
|
||||||
final AtomicInteger consumedCount = new AtomicInteger();
|
final AtomicInteger consumedCount = new AtomicInteger();
|
||||||
final CyclicBarrier barrier = new CyclicBarrier(2);
|
final CyclicBarrier barrier = new CyclicBarrier(2);
|
||||||
Thread producer = new Thread("producer")
|
ExecutorService producer = Executors.newCachedThreadPool();
|
||||||
{
|
|
||||||
public void run()
|
producer.submit(
|
||||||
{
|
new Runnable()
|
||||||
for (int i = 0; i < 2 * capacity; i++) {
|
{
|
||||||
final int taskID = i;
|
public void run()
|
||||||
System.out.println("Produced task"+ taskID);
|
{
|
||||||
executorService.submit(
|
for (int i = 0; i < 2 * capacity; i++) {
|
||||||
new Runnable()
|
final int taskID = i;
|
||||||
{
|
System.out.println("Produced task" + taskID);
|
||||||
@Override
|
executorService.submit(
|
||||||
public void run()
|
new Runnable()
|
||||||
{
|
{
|
||||||
System.out.println("Starting task" + taskID);
|
@Override
|
||||||
try {
|
public void run()
|
||||||
barrier.await();
|
{
|
||||||
|
System.out.println("Starting task" + taskID);
|
||||||
|
try {
|
||||||
|
consumedCount.incrementAndGet();
|
||||||
|
barrier.await(); //1
|
||||||
|
barrier.await(); //2
|
||||||
|
}
|
||||||
|
catch (Exception e) {
|
||||||
|
throw Throwables.propagate(e);
|
||||||
|
}
|
||||||
|
System.out.println("Completed task" + taskID);
|
||||||
|
}
|
||||||
}
|
}
|
||||||
catch (Exception e) {
|
);
|
||||||
throw Throwables.propagate(e);
|
producedCount.incrementAndGet();
|
||||||
}
|
}
|
||||||
consumedCount.incrementAndGet();
|
}
|
||||||
System.out.println("Completed task" + taskID);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
);
|
|
||||||
producedCount.incrementAndGet();
|
|
||||||
}
|
}
|
||||||
}
|
);
|
||||||
};
|
while(producedCount.get() < capacity ){
|
||||||
producer.start();
|
Thread.sleep(5);
|
||||||
|
}
|
||||||
|
|
||||||
for(int i=0;i<capacity;i++){
|
for(int i=0;i<capacity;i++){
|
||||||
Thread.sleep(500);
|
barrier.await(); //1
|
||||||
// total completed tasks +1 running task+ capacity = total produced tasks
|
// total consumed tasks + capacity = total produced tasks
|
||||||
Assert.assertEquals(consumedCount.intValue()+1+capacity,producedCount.intValue());
|
Assert.assertEquals(consumedCount.intValue() + capacity, producedCount.intValue());
|
||||||
barrier.await();
|
barrier.await(); //2
|
||||||
|
|
||||||
}
|
}
|
||||||
for(int i=0;i<capacity;i++){
|
for(int i=0;i<capacity;i++){
|
||||||
barrier.await();
|
barrier.await();
|
||||||
}
|
}
|
||||||
|
producer.shutdownNow();
|
||||||
|
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -479,7 +479,7 @@ public class RealtimePlumberSchool implements PlumberSchool
|
||||||
{
|
{
|
||||||
if (persistExecutor == null) {
|
if (persistExecutor == null) {
|
||||||
// use a blocking single threaded executor to throttle the firehose when write to disk is slow
|
// use a blocking single threaded executor to throttle the firehose when write to disk is slow
|
||||||
persistExecutor = Execs.blockingSingleThreaded(
|
persistExecutor = Execs.newBlockingSingleThreaded(
|
||||||
"plumber_persist_%d", maxPendingPersistBatches
|
"plumber_persist_%d", maxPendingPersistBatches
|
||||||
);
|
);
|
||||||
}
|
}
|
||||||
|
|
Loading…
Reference in New Issue