Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -334,8 +334,6 @@ public void stop()

queuedSize.set(0L);
failedAssignCount.set(0);
processingExecutor.shutdown();
callBackExecutor.shutdown();
}

private void entryRemoved(SegmentHolder segmentHolder, String path)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -73,7 +73,9 @@
import java.util.concurrent.ConcurrentMap;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.Executor;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicReference;

Expand Down Expand Up @@ -117,6 +119,9 @@ public class CuratorDruidCoordinatorTest extends CuratorTestBase
private final ObjectMapper jsonMapper;
private final ZkPathsConfig zkPathsConfig;

private ScheduledExecutorService peonExec = Execs.scheduledSingleThreaded("Master-PeonExec--%d");
private ExecutorService callbackExec = Execs.multiThreaded(4, "LoadQueuePeon-callbackexec--%d");

public CuratorDruidCoordinatorTest()
{
jsonMapper = TestHelper.makeJsonMapper();
Expand Down Expand Up @@ -186,16 +191,16 @@ public void setUp() throws Exception
curator,
SOURCE_LOAD_PATH,
objectMapper,
Execs.scheduledSingleThreaded("coordinator_test_load_queue_peon_src_scheduled-%d"),
Execs.singleThreaded("coordinator_test_load_queue_peon_src-%d"),
peonExec,
callbackExec,
druidCoordinatorConfig
);
destinationLoadQueuePeon = new CuratorLoadQueuePeon(
curator,
DESTINATION_LOAD_PATH,
objectMapper,
Execs.scheduledSingleThreaded("coordinator_test_load_queue_peon_dest_scheduled-%d"),
Execs.singleThreaded("coordinator_test_load_queue_peon_dest-%d"),
peonExec,
callbackExec,
druidCoordinatorConfig
);
druidNode = new DruidNode("hey", "what", false, 1234, null, true, false);
Expand Down Expand Up @@ -260,6 +265,15 @@ public void tearDown() throws Exception
@Rule
public final TestRule timeout = new DeadlockDetectingTimeout(60, TimeUnit.SECONDS);

@Test
public void testStopDoesntKillPoolItDoesntOwn() throws Exception
{
setupView();
sourceLoadQueuePeon.stop();
Assert.assertFalse(peonExec.isShutdown());
Assert.assertFalse(callbackExec.isShutdown());
}

@Test
public void testMoveSegment() throws Exception
{
Expand Down
12 changes: 9 additions & 3 deletions services/src/main/java/org/apache/druid/cli/CliCoordinator.java
Original file line number Diff line number Diff line change
Expand Up @@ -47,7 +47,9 @@
import org.apache.druid.guice.annotations.EscalatedGlobal;
import org.apache.druid.guice.http.JettyHttpClientModule;
import org.apache.druid.java.util.common.concurrent.Execs;
import org.apache.druid.java.util.common.concurrent.ExecutorServices;
import org.apache.druid.java.util.common.concurrent.ScheduledExecutorFactory;
import org.apache.druid.java.util.common.lifecycle.Lifecycle;
import org.apache.druid.java.util.common.logger.Logger;
import org.apache.druid.java.util.http.client.HttpClient;
import org.apache.druid.metadata.MetadataRuleManager;
Expand Down Expand Up @@ -248,17 +250,21 @@ public LoadQueueTaskMaster getLoadQueueTaskMaster(
ScheduledExecutorFactory factory,
DruidCoordinatorConfig config,
@EscalatedGlobal HttpClient httpClient,
ZkPathsConfig zkPaths
ZkPathsConfig zkPaths,
Lifecycle lifecycle
)
{
boolean useHttpLoadQueuePeon = "http".equalsIgnoreCase(config.getLoadQueuePeonType());
ExecutorService callBackExec;
if (useHttpLoadQueuePeon) {
callBackExec = Execs.singleThreaded("LoadQueuePeon-callbackexec--%d");
} else {
callBackExec = Execs.multiThreaded(config.getNumCuratorCallBackThreads(), "LoadQueuePeon"
+ "-callbackexec--%d");
callBackExec = Execs.multiThreaded(
config.getNumCuratorCallBackThreads(),
"LoadQueuePeon-callbackexec--%d"
);
}
ExecutorServices.manageLifecycle(lifecycle, callBackExec);
return new LoadQueueTaskMaster(
curator,
jsonMapper,
Expand Down