You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@druid.apache.org by GitBox <gi...@apache.org> on 2021/08/31 02:05:04 UTC

[GitHub] [druid] jihoonson opened a new pull request #11643: Cancel API for sqls

jihoonson opened a new pull request #11643:
URL: https://github.com/apache/druid/pull/11643


   ### Description
   
   This PR adds a cancellation API for SQLs mentioned in #4047. Similar to the cancellation API for native queries, this new API first performs authorization to see whether the requester has read permission for all resources of the query. If it does, the API cancels _all_ sql queries of the given `sqlQueryId`. 
   
   When a query is canceled, its `SqlLifecycle` is marked as canceled immediately. Once it's marked, any operation in `SqlLifecycle` becomes no-op so that we can avoid executing expensive operations unnecessarily. Additionally, I removed all locking in `SqlLifecycle` as it doesn't need to be thread-safe except for canceling. For canceling, it uses a volatile variable and a thread-safe data structure to avoid using locks in `SqlLifecycle`. This was necessary for the cancel API to be able to interrupt while the query is running, especially while executing `SqlLifecycle.execute()`.
   
   The web console is currently simply closing the client connection when the query is canceled and this behavior has not changed in this PR. As it seems that the web console is doing the same even for native queries, we can fix them all in a follow-up PR. Documentation will be in the follow-up PR as well.
   
   <hr>
   
   ##### Key changed/added classes in this PR
    * `SqlLifecycle`
    * `SqlLifecycleManager`
    * `SqlResource`
   
   <hr>
   
   <!-- Check the items by putting "x" in the brackets for the done things. Not all of these items apply to every PR. Remove the items which are not done or not relevant to the PR. None of the items from the checklist below are strictly necessary, but it would be very helpful if you at least self-review the PR. -->
   
   This PR has:
   - [x] been self-reviewed.
      - [ ] using the [concurrency checklist](https://github.com/apache/druid/blob/master/dev/code-review/concurrency.md) (Remove this item if the PR doesn't have any relation to concurrency.)
   - [ ] added documentation for new or modified features or behaviors.
   - [x] added Javadocs for most classes and all non-trivial methods. Linked related entities via Javadoc links.
   - [ ] added or updated version, license, or notice information in [licenses.yaml](https://github.com/apache/druid/blob/master/dev/license.md)
   - [ ] added comments explaining the "why" and the intent of the code wherever would not be obvious for an unfamiliar reader.
   - [x] added unit tests or modified existing tests to cover new code paths, ensuring the threshold for [code coverage](https://github.com/apache/druid/blob/master/dev/code-review/code-coverage.md) is met.
   - [x] added integration tests.
   - [x] been tested in a test Druid cluster.
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701492169



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,20 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {

Review comment:
       I added a new final state, `CANCELLED`.

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -372,15 +383,38 @@ public void after(boolean isDone, Throwable thrown)
   @VisibleForTesting
   public ValidationResult runAnalyzeResources(AuthenticationResult authenticationResult)
   {
-    synchronized (lock) {
-      return validate(authenticationResult);
+    return validate(authenticationResult);
+  }
+
+  public Set<Resource> getAuthorizedResources()
+  {
+    assert validationResult != null;
+    return validationResult.getResources();
+  }
+
+  /**
+   * Cancel all native queries associated to this lifecycle.
+   *
+   * This method is thread-safe.
+   */
+  public void cancel()
+  {
+    canceled = true;
+
+    final CopyOnWriteArrayList<String> nativeQueryIds = plannerContext.getNativeQueryIds();
+
+    for (String nativeQueryId : nativeQueryIds) {
+      log.debug("canceling native query [%s]", nativeQueryId);
+      queryScheduler.cancelQuery(nativeQueryId);
     }
   }
 
-  public RelDataType rowType()
+  public Optional<RelDataType> rowType()
   {
-    synchronized (lock) {
-      return plannerResult != null ? plannerResult.rowType() : prepareResult.getRowType();
+    if (canceled) {
+      return Optional.empty();

Review comment:
       > What you said about no-op only only seems true for the methods that transition lifecycle state. The methods which just get stuff, like `sqlQueryId()` don't seem to become no-ops and return empty when cancelled, which is why this seemed strange to me.
   
   This is more correct. I should have said state transition becomes no-op. I updated the comment in the code.
   
   > I guess this method is sort of strange because it actually expects to either `prepare()` or `plan()` to have been called, otherwise either `planResult` or `prepareResult` can be null (or it should null check it too). `DruidStatement` ensures that the statement is in the `PREPARED` state (it has different states than SqlLifecycle, which is fun) is in the correct state before calling this method, and `SqlResource` only calls it after it it has called `plan`, so checking for `null` `prepareResult` isn't strictly necessary, but might be nice to add.
   
   I agree this and other methods that I changed to return `Optional` seem strange. So, instead of that, I added `SqlRowTransformer` to do the necessary things without exposing stuffs in `SqlLifecycle` to outside. For this particular method of `rowType`, it is not used in JDBC as `DruidStatement` gets rowType directly from `PrepareResult` instead of `SqlLifecycle`. I think I would probably make `plan()` to return `PlannerResult` and do something similar if `PlannerResult` was as simple as `PrepareResult` and did not have `resultSupplier`. But since it does have `resultSupplier`, I think it's better to not expose it to outside but use `SqlRowTransformer`.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615062



##########
File path: integration-tests/src/test/java/org/apache/druid/tests/query/ITSqlCancelTest.java
##########
@@ -0,0 +1,143 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.tests.query;
+
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.google.common.collect.ImmutableMap;
+import com.google.inject.Inject;
+import org.apache.druid.java.util.common.ISE;
+import org.apache.druid.java.util.common.RE;
+import org.apache.druid.java.util.http.client.response.StatusResponseHolder;
+import org.apache.druid.query.QueryException;
+import org.apache.druid.query.QueryInterruptedException;
+import org.apache.druid.sql.http.SqlQuery;
+import org.apache.druid.testing.IntegrationTestingConfig;
+import org.apache.druid.testing.clients.CoordinatorResourceTestClient;
+import org.apache.druid.testing.clients.SqlResourceTestClient;
+import org.apache.druid.testing.guice.DruidTestModuleFactory;
+import org.apache.druid.testing.utils.ITRetryUtil;
+import org.apache.druid.testing.utils.SqlTestQueryHelper;
+import org.apache.druid.tests.TestNGGroup;
+import org.jboss.netty.handler.codec.http.HttpResponseStatus;
+import org.testng.annotations.BeforeMethod;
+import org.testng.annotations.Guice;
+import org.testng.annotations.Test;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+
+@Test(groups = TestNGGroup.QUERY)
+@Guice(moduleFactory = DruidTestModuleFactory.class)
+public class ITSqlCancelTest
+{
+  private static final String WIKIPEDIA_DATA_SOURCE = "wikipedia_editstream";
+
+  private static final String QUERY
+      = "SELECT sleep(CASE WHEN added > 0 THEN 1 ELSE 0 END) FROM wikipedia_editstream LIMIT 15";
+
+  private static final int NUM_QUERIES = 3;
+
+  @Inject
+  private CoordinatorResourceTestClient coordinatorClient;
+  @Inject
+  private SqlTestQueryHelper sqlHelper;
+  @Inject
+  private SqlResourceTestClient sqlClient;
+  @Inject
+  private IntegrationTestingConfig config;
+  @Inject
+  private ObjectMapper jsonMapper;
+
+  @BeforeMethod
+  public void before()
+  {
+    // ensure that wikipedia segments are loaded completely
+    ITRetryUtil.retryUntilTrue(
+        () -> coordinatorClient.areSegmentsLoaded(WIKIPEDIA_DATA_SOURCE), "wikipedia segment load"
+    );
+  }
+
+  @Test
+  public void testCancelValidQuery() throws Exception
+  {
+    final String queryId = "sql-cancel-test";
+    final List<Future<StatusResponseHolder>> queryResponseFutures = new ArrayList<>();
+    for (int i = 0; i < NUM_QUERIES; i++) {
+      queryResponseFutures.add(
+          sqlClient.queryAsync(
+              sqlHelper.getQueryURL(config.getRouterUrl()),
+              new SqlQuery(QUERY, null, false, ImmutableMap.of("sqlQueryId", queryId), null)
+          )
+      );
+    }
+
+    // Wait until the sqlLifecycle is authorized and registered
+    Thread.sleep(1000);

Review comment:
       One case it could be flaky is that the test query finishes in less than one second. I changed the query to run exactly for 15 seconds and added a comment about it.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615404



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,22 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {
+      throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
+    }
+    final DruidPlanner planner0;

Review comment:
       Good catch. I think they remained unreverted after I reverted some unnecessary changes.

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -304,12 +301,13 @@ public PrepareResult prepare() throws RelConversionException
    *
    * If successful, the lifecycle will first transition from {@link State#AUTHORIZED} to {@link State#PLANNED}.
    */
-  public PlannerContext plan() throws RelConversionException
+  public void plan() throws RelConversionException
   {
-    synchronized (lock) {
-      transition(State.AUTHORIZED, State.PLANNED);
+    if (transition(State.AUTHORIZED, State.PLANNED)) {
+      final DruidPlanner planner0;
       Preconditions.checkNotNull(plannerContext, "Cannot plan, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
+      planner0 = plannerFactory.createPlannerWithContext(plannerContext);
+      try (DruidPlanner planner = planner0) {

Review comment:
       Reverted.

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycleManager.java
##########
@@ -0,0 +1,98 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import com.google.common.collect.ImmutableList;
+import com.google.errorprone.annotations.concurrent.GuardedBy;
+import org.apache.druid.guice.LazySingleton;
+import org.apache.druid.sql.SqlLifecycle.State;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+/**
+ * This class manages only _authorized_ {@link SqlLifecycle}s. The main use case of this class is
+ * tracking running queries so that the cancel API can identify the lifecycles to cancel.
+ *
+ * This class is thread-safe as there are 2 or more threads that can access lifecycles at the same time
+ * for query running or query canceling.
+ */
+@LazySingleton
+public class SqlLifecycleManager
+{
+  private final Object lock = new Object();
+
+  @GuardedBy("lock")
+  private final Map<String, List<SqlLifecycle>> sqlLifecycles = new HashMap<>();
+
+  public void add(String sqlQueryId, SqlLifecycle lifecycle)
+  {
+    synchronized (lock) {
+      assert lifecycle.getState().ordinal() == State.AUTHORIZED.ordinal();
+      sqlLifecycles.computeIfAbsent(sqlQueryId, k -> new ArrayList<>())
+                   .add(lifecycle);
+    }
+  }
+
+  /**
+   * Removes the given lifecycle of the given query ID.
+   * This method uses {@link Object#equals} to find the lifecycle matched to the given parameter.
+   */
+  public void remove(String sqlQueryId, SqlLifecycle lifecycle)
+  {
+    synchronized (lock) {
+      List<SqlLifecycle> lifecycles = sqlLifecycles.get(sqlQueryId);
+      if (lifecycles != null) {
+        lifecycles.remove(lifecycle);
+        if (lifecycles.isEmpty()) {
+          sqlLifecycles.remove(sqlQueryId);
+        }
+      }
+    }
+  }
+
+  /**
+   * Removes all lifecycles of the given query ID.

Review comment:
       Fixed javadoc.

##########
File path: sql/src/main/java/org/apache/druid/sql/http/SqlResource.java
##########
@@ -182,27 +220,28 @@ public Response doPost(
       }
     }
     catch (QueryCapacityExceededException cap) {
-      lifecycle.emitLogsAndMetrics(cap, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, cap, remoteAddr, -1);
       return buildNonOkResponse(QueryCapacityExceededException.STATUS_CODE, cap);
     }
     catch (QueryUnsupportedException unsupported) {
-      lifecycle.emitLogsAndMetrics(unsupported, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, unsupported, remoteAddr, -1);
       return buildNonOkResponse(QueryUnsupportedException.STATUS_CODE, unsupported);
     }
     catch (QueryTimeoutException timeout) {
-      lifecycle.emitLogsAndMetrics(timeout, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, timeout, remoteAddr, -1);
       return buildNonOkResponse(QueryTimeoutException.STATUS_CODE, timeout);
     }
     catch (SqlPlanningException | ResourceLimitExceededException e) {
-      lifecycle.emitLogsAndMetrics(e, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, e, remoteAddr, -1);
       return buildNonOkResponse(BadQueryException.STATUS_CODE, e);
     }
     catch (ForbiddenException e) {
+      sqlLifecycleManager.remove(sqlQueryId, lifecycle);

Review comment:
       I added `endLifecycleWithoutEmittingMetrics`.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615601



##########
File path: sql/src/main/java/org/apache/druid/sql/http/SqlResource.java
##########
@@ -222,11 +261,70 @@ public Response doPost(
     }
   }
 
-  Response buildNonOkResponse(int status, Exception e) throws JsonProcessingException
+  private void endLifecycle(
+      String sqlQueryId,
+      SqlLifecycle lifecycle,
+      @Nullable final Throwable e,
+      @Nullable final String remoteAddress,
+      final long bytesWritten
+  )
+  {
+    lifecycle.emitLogsAndMetrics(e, remoteAddress, bytesWritten);
+    sqlLifecycleManager.remove(sqlQueryId, lifecycle);
+  }
+
+  private Response buildCanceledResponse(String sqlQueryId) throws JsonProcessingException
+  {
+    return buildNonOkResponse(
+        Status.INTERNAL_SERVER_ERROR.getStatusCode(),
+        new QueryInterruptedException(
+            QueryInterruptedException.QUERY_CANCELLED,
+            StringUtils.format("Query is canceled [%s]", sqlQueryId),
+            null,
+            null
+        )
+    );
+  }
+
+  private Response buildNonOkResponse(int status, Exception e) throws JsonProcessingException
   {
     return Response.status(status)
                    .type(MediaType.APPLICATION_JSON_TYPE)
                    .entity(jsonMapper.writeValueAsBytes(e))
                    .build();
   }
+
+  @DELETE
+  @Path("{id}")
+  @Produces(MediaType.APPLICATION_JSON)
+  public Response cancelQuery(
+      @PathParam("id") String sqlQueryId,
+      @Context final HttpServletRequest req
+  )
+  {
+    log.debug("Received cancel request for query [%s]", sqlQueryId);
+
+    List<SqlLifecycle> lifecycles = sqlLifecycleManager.getAll(sqlQueryId);
+    if (lifecycles.isEmpty()) {
+      return Response.status(Status.NOT_FOUND).build();
+    }
+    Set<Resource> resources = lifecycles
+        .stream()
+        .flatMap(lifecycle -> lifecycle.getAuthorizedResources().stream())
+        .collect(Collectors.toSet());
+    Access access = AuthorizationUtils.authorizeAllResourceActions(
+        req,
+        Iterables.transform(resources, AuthorizationUtils.RESOURCE_READ_RA_GENERATOR),
+        authorizerMapper
+    );
+
+    if (access.isAllowed()) {
+      sqlLifecycleManager.removeAll(sqlQueryId, lifecycles);

Review comment:
       That case can happen and it is intentional to not cancel those queries. This should remove only the lifecycles in the snapshot. After all, you cannot cancel future queries that can be issued after the cancel request is processed.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] maytasm commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
maytasm commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r699998346



##########
File path: integration-tests/src/test/java/org/apache/druid/tests/query/ITSqlCancelTest.java
##########
@@ -0,0 +1,143 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.tests.query;
+
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.google.common.collect.ImmutableMap;
+import com.google.inject.Inject;
+import org.apache.druid.java.util.common.ISE;
+import org.apache.druid.java.util.common.RE;
+import org.apache.druid.java.util.http.client.response.StatusResponseHolder;
+import org.apache.druid.query.QueryException;
+import org.apache.druid.query.QueryInterruptedException;
+import org.apache.druid.sql.http.SqlQuery;
+import org.apache.druid.testing.IntegrationTestingConfig;
+import org.apache.druid.testing.clients.CoordinatorResourceTestClient;
+import org.apache.druid.testing.clients.SqlResourceTestClient;
+import org.apache.druid.testing.guice.DruidTestModuleFactory;
+import org.apache.druid.testing.utils.ITRetryUtil;
+import org.apache.druid.testing.utils.SqlTestQueryHelper;
+import org.apache.druid.tests.TestNGGroup;
+import org.jboss.netty.handler.codec.http.HttpResponseStatus;
+import org.testng.annotations.BeforeMethod;
+import org.testng.annotations.Guice;
+import org.testng.annotations.Test;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+
+@Test(groups = TestNGGroup.QUERY)
+@Guice(moduleFactory = DruidTestModuleFactory.class)
+public class ITSqlCancelTest
+{
+  private static final String WIKIPEDIA_DATA_SOURCE = "wikipedia_editstream";
+
+  private static final String QUERY
+      = "SELECT sleep(CASE WHEN added > 0 THEN 1 ELSE 0 END) FROM wikipedia_editstream LIMIT 15";
+
+  private static final int NUM_QUERIES = 3;
+
+  @Inject
+  private CoordinatorResourceTestClient coordinatorClient;
+  @Inject
+  private SqlTestQueryHelper sqlHelper;
+  @Inject
+  private SqlResourceTestClient sqlClient;
+  @Inject
+  private IntegrationTestingConfig config;
+  @Inject
+  private ObjectMapper jsonMapper;
+
+  @BeforeMethod
+  public void before()
+  {
+    // ensure that wikipedia segments are loaded completely
+    ITRetryUtil.retryUntilTrue(
+        () -> coordinatorClient.areSegmentsLoaded(WIKIPEDIA_DATA_SOURCE), "wikipedia segment load"
+    );
+  }
+
+  @Test
+  public void testCancelValidQuery() throws Exception
+  {
+    final String queryId = "sql-cancel-test";
+    final List<Future<StatusResponseHolder>> queryResponseFutures = new ArrayList<>();
+    for (int i = 0; i < NUM_QUERIES; i++) {
+      queryResponseFutures.add(
+          sqlClient.queryAsync(
+              sqlHelper.getQueryURL(config.getRouterUrl()),
+              new SqlQuery(QUERY, null, false, ImmutableMap.of("sqlQueryId", queryId), null)
+          )
+      );
+    }
+
+    // Wait until the sqlLifecycle is authorized and registered
+    Thread.sleep(1000);

Review comment:
       Is there any way to make this more deterministic? Could this cause test to be flaky?

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -304,12 +301,13 @@ public PrepareResult prepare() throws RelConversionException
    *
    * If successful, the lifecycle will first transition from {@link State#AUTHORIZED} to {@link State#PLANNED}.
    */
-  public PlannerContext plan() throws RelConversionException
+  public void plan() throws RelConversionException
   {
-    synchronized (lock) {
-      transition(State.AUTHORIZED, State.PLANNED);
+    if (transition(State.AUTHORIZED, State.PLANNED)) {
+      final DruidPlanner planner0;
       Preconditions.checkNotNull(plannerContext, "Cannot plan, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
+      planner0 = plannerFactory.createPlannerWithContext(plannerContext);
+      try (DruidPlanner planner = planner0) {

Review comment:
       nit: Why the refactor from `try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext))` ?

##########
File path: sql/src/main/java/org/apache/druid/sql/http/SqlResource.java
##########
@@ -182,27 +220,28 @@ public Response doPost(
       }
     }
     catch (QueryCapacityExceededException cap) {
-      lifecycle.emitLogsAndMetrics(cap, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, cap, remoteAddr, -1);
       return buildNonOkResponse(QueryCapacityExceededException.STATUS_CODE, cap);
     }
     catch (QueryUnsupportedException unsupported) {
-      lifecycle.emitLogsAndMetrics(unsupported, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, unsupported, remoteAddr, -1);
       return buildNonOkResponse(QueryUnsupportedException.STATUS_CODE, unsupported);
     }
     catch (QueryTimeoutException timeout) {
-      lifecycle.emitLogsAndMetrics(timeout, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, timeout, remoteAddr, -1);
       return buildNonOkResponse(QueryTimeoutException.STATUS_CODE, timeout);
     }
     catch (SqlPlanningException | ResourceLimitExceededException e) {
-      lifecycle.emitLogsAndMetrics(e, remoteAddr, -1);
+      endLifecycle(sqlQueryId, lifecycle, e, remoteAddr, -1);
       return buildNonOkResponse(BadQueryException.STATUS_CODE, e);
     }
     catch (ForbiddenException e) {
+      sqlLifecycleManager.remove(sqlQueryId, lifecycle);

Review comment:
       nit: maybe add a boolean argument for endLifecycle to emit metric or not and call endLifecycle here too?

##########
File path: sql/src/main/java/org/apache/druid/sql/avatica/DruidStatement.java
##########
@@ -216,7 +218,9 @@ public DruidStatement execute(List<TypedValue> parameters)
         sqlLifecycle.setParameters(parameters);
         sqlLifecycle.validateAndAuthorize(authenticationResult);
         sqlLifecycle.plan();
-        final Sequence<Object[]> baseSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();
+        Optional<Sequence<Object[]>> maybeSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();
+        assert maybeSequence.isPresent();

Review comment:
       Can this be not present?

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -80,8 +84,7 @@
  * <li>Logging ({@link #emitLogsAndMetrics(Throwable, String, long)})</li>
  * </ol>
  *
- * <p>Unlike QueryLifecycle, this class is designed to be <b>thread safe</b> so that it can be used in multi-threaded
- * scenario (JDBC) without external synchronization.
+ * Every method in this class must be called by the same thread except for {@link #cancel()}.

Review comment:
       What's the reasoning for removing the thread safe behavior? and will there be any problem with multi-threaded scenario (JDBC) mentioned in the old javadoc?

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,22 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {
+      throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
+    }
+    final DruidPlanner planner0;

Review comment:
       nit: Why the refactor from `try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext))` ?

##########
File path: sql/src/main/java/org/apache/druid/sql/http/SqlResource.java
##########
@@ -222,11 +261,70 @@ public Response doPost(
     }
   }
 
-  Response buildNonOkResponse(int status, Exception e) throws JsonProcessingException
+  private void endLifecycle(
+      String sqlQueryId,
+      SqlLifecycle lifecycle,
+      @Nullable final Throwable e,
+      @Nullable final String remoteAddress,
+      final long bytesWritten
+  )
+  {
+    lifecycle.emitLogsAndMetrics(e, remoteAddress, bytesWritten);
+    sqlLifecycleManager.remove(sqlQueryId, lifecycle);
+  }
+
+  private Response buildCanceledResponse(String sqlQueryId) throws JsonProcessingException
+  {
+    return buildNonOkResponse(
+        Status.INTERNAL_SERVER_ERROR.getStatusCode(),
+        new QueryInterruptedException(
+            QueryInterruptedException.QUERY_CANCELLED,
+            StringUtils.format("Query is canceled [%s]", sqlQueryId),
+            null,
+            null
+        )
+    );
+  }
+
+  private Response buildNonOkResponse(int status, Exception e) throws JsonProcessingException
   {
     return Response.status(status)
                    .type(MediaType.APPLICATION_JSON_TYPE)
                    .entity(jsonMapper.writeValueAsBytes(e))
                    .build();
   }
+
+  @DELETE
+  @Path("{id}")
+  @Produces(MediaType.APPLICATION_JSON)
+  public Response cancelQuery(
+      @PathParam("id") String sqlQueryId,
+      @Context final HttpServletRequest req
+  )
+  {
+    log.debug("Received cancel request for query [%s]", sqlQueryId);
+
+    List<SqlLifecycle> lifecycles = sqlLifecycleManager.getAll(sqlQueryId);
+    if (lifecycles.isEmpty()) {
+      return Response.status(Status.NOT_FOUND).build();
+    }
+    Set<Resource> resources = lifecycles
+        .stream()
+        .flatMap(lifecycle -> lifecycle.getAuthorizedResources().stream())
+        .collect(Collectors.toSet());
+    Access access = AuthorizationUtils.authorizeAllResourceActions(
+        req,
+        Iterables.transform(resources, AuthorizationUtils.RESOURCE_READ_RA_GENERATOR),
+        authorizerMapper
+    );
+
+    if (access.isAllowed()) {
+      sqlLifecycleManager.removeAll(sqlQueryId, lifecycles);

Review comment:
       The sqlQueryId should have no lifecycle left right? Is it possible that the cancel happens when not **all** lifecycle are authorized and added to the manager yet?

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycleManager.java
##########
@@ -0,0 +1,98 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import com.google.common.collect.ImmutableList;
+import com.google.errorprone.annotations.concurrent.GuardedBy;
+import org.apache.druid.guice.LazySingleton;
+import org.apache.druid.sql.SqlLifecycle.State;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+/**
+ * This class manages only _authorized_ {@link SqlLifecycle}s. The main use case of this class is
+ * tracking running queries so that the cancel API can identify the lifecycles to cancel.
+ *
+ * This class is thread-safe as there are 2 or more threads that can access lifecycles at the same time
+ * for query running or query canceling.
+ */
+@LazySingleton
+public class SqlLifecycleManager
+{
+  private final Object lock = new Object();
+
+  @GuardedBy("lock")
+  private final Map<String, List<SqlLifecycle>> sqlLifecycles = new HashMap<>();
+
+  public void add(String sqlQueryId, SqlLifecycle lifecycle)
+  {
+    synchronized (lock) {
+      assert lifecycle.getState().ordinal() == State.AUTHORIZED.ordinal();
+      sqlLifecycles.computeIfAbsent(sqlQueryId, k -> new ArrayList<>())
+                   .add(lifecycle);
+    }
+  }
+
+  /**
+   * Removes the given lifecycle of the given query ID.
+   * This method uses {@link Object#equals} to find the lifecycle matched to the given parameter.
+   */
+  public void remove(String sqlQueryId, SqlLifecycle lifecycle)
+  {
+    synchronized (lock) {
+      List<SqlLifecycle> lifecycles = sqlLifecycles.get(sqlQueryId);
+      if (lifecycles != null) {
+        lifecycles.remove(lifecycle);
+        if (lifecycles.isEmpty()) {
+          sqlLifecycles.remove(sqlQueryId);
+        }
+      }
+    }
+  }
+
+  /**
+   * Removes all lifecycles of the given query ID.

Review comment:
       This javadoc should mention that only lifecycles matching the lifecycles in lifecyclesToRemove is removed




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700118000



##########
File path: services/src/main/java/org/apache/druid/server/AsyncQueryForwardingServlet.java
##########
@@ -286,7 +286,7 @@ protected void service(HttpServletRequest request, HttpServletResponse response)
   private void broadcastQueryCancelRequest(HttpServletRequest request, Server targetServer)
   {
     // send query cancellation to all brokers this query may have gone to
-    // to keep the code simple, the proxy servlet will also send a request to the default targetServer.
+    // keep the code simple, the proxy servlet will also send a request to the default targetServer.

Review comment:
       these comments just seem to be repeating what the javadoc says, maybe just remove them completely?

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -372,15 +383,38 @@ public void after(boolean isDone, Throwable thrown)
   @VisibleForTesting
   public ValidationResult runAnalyzeResources(AuthenticationResult authenticationResult)
   {
-    synchronized (lock) {
-      return validate(authenticationResult);
+    return validate(authenticationResult);
+  }
+
+  public Set<Resource> getAuthorizedResources()
+  {
+    assert validationResult != null;
+    return validationResult.getResources();
+  }
+
+  /**
+   * Cancel all native queries associated to this lifecycle.
+   *
+   * This method is thread-safe.
+   */
+  public void cancel()
+  {
+    canceled = true;
+
+    final CopyOnWriteArrayList<String> nativeQueryIds = plannerContext.getNativeQueryIds();
+
+    for (String nativeQueryId : nativeQueryIds) {
+      log.debug("canceling native query [%s]", nativeQueryId);
+      queryScheduler.cancelQuery(nativeQueryId);
     }
   }
 
-  public RelDataType rowType()
+  public Optional<RelDataType> rowType()
   {
-    synchronized (lock) {
-      return plannerResult != null ? plannerResult.rowType() : prepareResult.getRowType();
+    if (canceled) {
+      return Optional.empty();

Review comment:
       why would being cancelled affect the row type?




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701561945



##########
File path: .idea/misc.xml
##########
@@ -84,7 +84,7 @@
     <resource url="http://maven.apache.org/ASSEMBLY/2.0.0" location="$PROJECT_DIR$/.idea/xml-schemas/assembly-2.0.0.xsd" />
     <resource url="http://www.w3.org/Graphics/SVG/1.1/DTD/svg11.dtd" location="$PROJECT_DIR$/.idea/xml-schemas/svg11.dtd" />
   </component>
-  <component name="ProjectRootManager" version="2" languageLevel="JDK_1_8" default="false" project-jdk-name="1.8" project-jdk-type="JavaSDK">
+  <component name="ProjectRootManager" version="2" languageLevel="JDK_1_8" project-jdk-name="azul-11" project-jdk-type="JavaSDK">

Review comment:
       Oops, I was testing a flaky test which failed only with java 11. Reverted.

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlRowTransformer.java
##########
@@ -0,0 +1,81 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import org.apache.calcite.rel.type.RelDataType;
+import org.apache.calcite.sql.type.SqlTypeName;
+import org.apache.druid.sql.calcite.planner.Calcites;
+import org.joda.time.DateTimeZone;
+import org.joda.time.format.ISODateTimeFormat;
+
+import javax.annotation.Nullable;
+import java.util.ArrayList;
+import java.util.List;
+
+/**
+ * This class transforms the values of TIMESTAMP or DATE type for sql query results.
+ * The transformation is required only when the sql query is submitted to {@link org.apache.druid.sql.http.SqlResource}.
+ */
+public class SqlRowTransformer

Review comment:
       Yeah I think we should consolidate them someday. I didn't do it in this PR to avoid invasive refactoring.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615322



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -80,8 +84,7 @@
  * <li>Logging ({@link #emitLogsAndMetrics(Throwable, String, long)})</li>
  * </ol>
  *
- * <p>Unlike QueryLifecycle, this class is designed to be <b>thread safe</b> so that it can be used in multi-threaded
- * scenario (JDBC) without external synchronization.
+ * Every method in this class must be called by the same thread except for {@link #cancel()}.

Review comment:
       That javadoc seems no longer valid. `SqlLifecycle` is always used under a lock in `DruidStatement` which is the same lock used in `DruidStatement.nextFrame()`. I believe locking in `SqlLifecycle` is redundant except for sql canceling. Meanwhile, it was necessary to get rid of locking in `SqlLifecycle`, especially in `execute()` to be able to cancel the query while its execution.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700616127



##########
File path: services/src/main/java/org/apache/druid/server/AsyncQueryForwardingServlet.java
##########
@@ -286,7 +286,7 @@ protected void service(HttpServletRequest request, HttpServletResponse response)
   private void broadcastQueryCancelRequest(HttpServletRequest request, Server targetServer)
   {
     // send query cancellation to all brokers this query may have gone to
-    // to keep the code simple, the proxy servlet will also send a request to the default targetServer.
+    // keep the code simple, the proxy servlet will also send a request to the default targetServer.

Review comment:
       Removed the comment.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on pull request #11643:
URL: https://github.com/apache/druid/pull/11643#issuecomment-913198723


   @maytasm @clintropolis thank you for the review.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701392151



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,20 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {

Review comment:
       Oh, i guess if `CANCELLED` was a real state you would need the locking on state which might complicate things a bit in a different way than cancel being a standalone thing, though maybe not much since I think only `transition` would need to be synchronized.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701509642



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -372,15 +383,38 @@ public void after(boolean isDone, Throwable thrown)
   @VisibleForTesting
   public ValidationResult runAnalyzeResources(AuthenticationResult authenticationResult)
   {
-    synchronized (lock) {
-      return validate(authenticationResult);
+    return validate(authenticationResult);
+  }
+
+  public Set<Resource> getAuthorizedResources()
+  {
+    assert validationResult != null;
+    return validationResult.getResources();
+  }
+
+  /**
+   * Cancel all native queries associated to this lifecycle.
+   *
+   * This method is thread-safe.
+   */
+  public void cancel()
+  {
+    canceled = true;
+
+    final CopyOnWriteArrayList<String> nativeQueryIds = plannerContext.getNativeQueryIds();
+
+    for (String nativeQueryId : nativeQueryIds) {
+      log.debug("canceling native query [%s]", nativeQueryId);
+      queryScheduler.cancelQuery(nativeQueryId);
     }
   }
 
-  public RelDataType rowType()
+  public Optional<RelDataType> rowType()
   {
-    synchronized (lock) {
-      return plannerResult != null ? plannerResult.rowType() : prepareResult.getRowType();
+    if (canceled) {
+      return Optional.empty();

Review comment:
       I changed the behavior of transition being no-op when the query is cancelled to exploding immediately with `QueryInterruptedException`. This will let the query emit a request log and metrics even when the query is cancelled before its execution. I updated the PR description to include this change.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615711



##########
File path: sql/src/main/java/org/apache/druid/sql/avatica/DruidStatement.java
##########
@@ -216,7 +218,9 @@ public DruidStatement execute(List<TypedValue> parameters)
         sqlLifecycle.setParameters(parameters);
         sqlLifecycle.validateAndAuthorize(authenticationResult);
         sqlLifecycle.plan();
-        final Sequence<Object[]> baseSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();
+        Optional<Sequence<Object[]>> maybeSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();
+        assert maybeSequence.isPresent();

Review comment:
       It's not possible since `sqlLifecycle` is not registered in `SqlLifecycleManager` and thus not cancellable. I added a comment about it.  




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson merged pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson merged pull request #11643:
URL: https://github.com/apache/druid/pull/11643


   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700107282



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycleManager.java
##########
@@ -0,0 +1,98 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import com.google.common.collect.ImmutableList;
+import com.google.errorprone.annotations.concurrent.GuardedBy;
+import org.apache.druid.guice.LazySingleton;
+import org.apache.druid.sql.SqlLifecycle.State;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+/**
+ * This class manages only _authorized_ {@link SqlLifecycle}s. The main use case of this class is
+ * tracking running queries so that the cancel API can identify the lifecycles to cancel.
+ *
+ * This class is thread-safe as there are 2 or more threads that can access lifecycles at the same time
+ * for query running or query canceling.
+ */
+@LazySingleton
+public class SqlLifecycleManager

Review comment:
       it seems a shame to me that this has to exist and we can't instead improve `QueryScheduler` to allow `SqlLifecycle` to register cancellables (with cascading to native queryIds) and finishers to cleanup similar to what it does already for native queries, the reasoning being to consolidate cancellation logic in one place instead of it being spread around like it is right now. I'm not quite sure what exactly this looks like, but maybe less `Future` centric. The fork join pool parallel merge sequences could similarly be wired up to more proactively cancel merge tasks and also aren't future based, though I have prototyped at some point in the past just wrapping it in a Future to fit the interface.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701387477



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -372,15 +383,38 @@ public void after(boolean isDone, Throwable thrown)
   @VisibleForTesting
   public ValidationResult runAnalyzeResources(AuthenticationResult authenticationResult)
   {
-    synchronized (lock) {
-      return validate(authenticationResult);
+    return validate(authenticationResult);
+  }
+
+  public Set<Resource> getAuthorizedResources()
+  {
+    assert validationResult != null;
+    return validationResult.getResources();
+  }
+
+  /**
+   * Cancel all native queries associated to this lifecycle.
+   *
+   * This method is thread-safe.
+   */
+  public void cancel()
+  {
+    canceled = true;
+
+    final CopyOnWriteArrayList<String> nativeQueryIds = plannerContext.getNativeQueryIds();
+
+    for (String nativeQueryId : nativeQueryIds) {
+      log.debug("canceling native query [%s]", nativeQueryId);
+      queryScheduler.cancelQuery(nativeQueryId);
     }
   }
 
-  public RelDataType rowType()
+  public Optional<RelDataType> rowType()
   {
-    synchronized (lock) {
-      return plannerResult != null ? plannerResult.rowType() : prepareResult.getRowType();
+    if (canceled) {
+      return Optional.empty();

Review comment:
       What you said about no-op only only seems true for the methods that transition lifecycle state. The methods which just get stuff, like `sqlQueryId()` don't seem to become no-ops and return empty when cancelled, which is why this seemed strange to me.
   
   I guess this method is sort of strange because it actually expects to either `prepare()` or `plan()` to have been called, otherwise either `planResult` or `prepareResult` can be null (or it should null check it too). `DruidStatement` ensures that the statement is in the `PREPARED` state (it has different states than SqlLifecycle, which is fun) is in the correct state before calling this method, and `SqlResource` only calls it after it it has called `plan`, so checking for `null` `prepareResult` isn't strictly necessary, but might be nice to add.
   
   

##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,20 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {

Review comment:
       since `prepare` doesn't transition state, it might be nice to check for cancelled here so we don't do silly stuff.
   
   I guess alternatively we could have added `State.CANCELLED`, since then all of the state transitions would naturally fail because the expected previous state would not match expectations and `transition` wouldn't have to be a boolean that callers check. This was seems ok too though, so up to you if you want to make `CANCELLED` a real state or not.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700616052



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -372,15 +383,38 @@ public void after(boolean isDone, Throwable thrown)
   @VisibleForTesting
   public ValidationResult runAnalyzeResources(AuthenticationResult authenticationResult)
   {
-    synchronized (lock) {
-      return validate(authenticationResult);
+    return validate(authenticationResult);
+  }
+
+  public Set<Resource> getAuthorizedResources()
+  {
+    assert validationResult != null;
+    return validationResult.getResources();
+  }
+
+  /**
+   * Cancel all native queries associated to this lifecycle.
+   *
+   * This method is thread-safe.
+   */
+  public void cancel()
+  {
+    canceled = true;
+
+    final CopyOnWriteArrayList<String> nativeQueryIds = plannerContext.getNativeQueryIds();
+
+    for (String nativeQueryId : nativeQueryIds) {
+      log.debug("canceling native query [%s]", nativeQueryId);
+      queryScheduler.cancelQuery(nativeQueryId);
     }
   }
 
-  public RelDataType rowType()
+  public Optional<RelDataType> rowType()
   {
-    synchronized (lock) {
-      return plannerResult != null ? plannerResult.rowType() : prepareResult.getRowType();
+    if (canceled) {
+      return Optional.empty();

Review comment:
       The methods in `SqlLifecycle` becomes no-op once it's cancelled. As a result, `plannerResult` can be null if it was cancelled before `plan()`.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701546354



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlRowTransformer.java
##########
@@ -0,0 +1,81 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import org.apache.calcite.rel.type.RelDataType;
+import org.apache.calcite.sql.type.SqlTypeName;
+import org.apache.druid.sql.calcite.planner.Calcites;
+import org.joda.time.DateTimeZone;
+import org.joda.time.format.ISODateTimeFormat;
+
+import javax.annotation.Nullable;
+import java.util.ArrayList;
+import java.util.List;
+
+/**
+ * This class transforms the values of TIMESTAMP or DATE type for sql query results.
+ * The transformation is required only when the sql query is submitted to {@link org.apache.druid.sql.http.SqlResource}.
+ */
+public class SqlRowTransformer

Review comment:
       nice :+1: I wonder if someday we should consolidate the transformation that is done here, with the stuff [`QueryMaker` is doing](https://github.com/apache/druid/blob/master/sql/src/main/java/org/apache/druid/sql/calcite/rel/QueryMaker.java#L165), like maybe this transformer could be pushed all the way down, and maybe handle [the special stuff it is doing for JDBC there the same way with a transformer instead]( https://github.com/apache/druid/blob/master/sql/src/main/java/org/apache/druid/sql/calcite/rel/QueryMaker.java#L332)

##########
File path: .idea/misc.xml
##########
@@ -84,7 +84,7 @@
     <resource url="http://maven.apache.org/ASSEMBLY/2.0.0" location="$PROJECT_DIR$/.idea/xml-schemas/assembly-2.0.0.xsd" />
     <resource url="http://www.w3.org/Graphics/SVG/1.1/DTD/svg11.dtd" location="$PROJECT_DIR$/.idea/xml-schemas/svg11.dtd" />
   </component>
-  <component name="ProjectRootManager" version="2" languageLevel="JDK_1_8" default="false" project-jdk-name="1.8" project-jdk-type="JavaSDK">
+  <component name="ProjectRootManager" version="2" languageLevel="JDK_1_8" project-jdk-name="azul-11" project-jdk-type="JavaSDK">

Review comment:
       unintended change?

##########
File path: sql/src/main/java/org/apache/druid/sql/avatica/DruidStatement.java
##########
@@ -216,7 +218,7 @@ public DruidStatement execute(List<TypedValue> parameters)
         sqlLifecycle.setParameters(parameters);
         sqlLifecycle.validateAndAuthorize(authenticationResult);
         sqlLifecycle.plan();
-        final Sequence<Object[]> baseSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();
+        Sequence<Object[]> baseSequence = yielderOpenCloseExecutor.submit(sqlLifecycle::execute).get();

Review comment:
       nit: why not final?




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] clintropolis commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
clintropolis commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r701392151



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycle.java
##########
@@ -280,22 +277,20 @@ private void checkAccess(Access access)
    */
   public PrepareResult prepare() throws RelConversionException
   {
-    synchronized (lock) {
-      if (state != State.AUTHORIZED) {
-        throw new ISE("Cannot prepare because current state[%s] is not [%s].", state, State.AUTHORIZED);
-      }
-      Preconditions.checkNotNull(plannerContext, "Cannot prepare, plannerContext is null");
-      try (DruidPlanner planner = plannerFactory.createPlannerWithContext(plannerContext)) {
-        this.prepareResult = planner.prepare(sql);
-        return prepareResult;
-      }
-      // we can't collapse catch clauses since SqlPlanningException has type-sensitive constructors.
-      catch (SqlParseException e) {
-        throw new SqlPlanningException(e);
-      }
-      catch (ValidationException e) {
-        throw new SqlPlanningException(e);
-      }
+    if (state != State.AUTHORIZED) {

Review comment:
       Oh, i guess if `CANCELLED` was a real state you would need the locking on state which might complicate things a bit in a different way than cancel being a standalone thing.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson commented on a change in pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson commented on a change in pull request #11643:
URL: https://github.com/apache/druid/pull/11643#discussion_r700615920



##########
File path: sql/src/main/java/org/apache/druid/sql/SqlLifecycleManager.java
##########
@@ -0,0 +1,98 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.druid.sql;
+
+import com.google.common.collect.ImmutableList;
+import com.google.errorprone.annotations.concurrent.GuardedBy;
+import org.apache.druid.guice.LazySingleton;
+import org.apache.druid.sql.SqlLifecycle.State;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+/**
+ * This class manages only _authorized_ {@link SqlLifecycle}s. The main use case of this class is
+ * tracking running queries so that the cancel API can identify the lifecycles to cancel.
+ *
+ * This class is thread-safe as there are 2 or more threads that can access lifecycles at the same time
+ * for query running or query canceling.
+ */
+@LazySingleton
+public class SqlLifecycleManager

Review comment:
       I see your point, but it seemed like a natural choice to me to add this new class instead of using `QueryScheduler` since `QueryScheduler` is currently designed to manage only native queries. It also has richer functionalities to manage cluster recourse based on query lanes. It doesn't seem like there is a strong reason for merging them at least at this point. Instead of merging them, I added some javadoc about how they are different today, so that our future selves can remember and make a better decision.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org


[GitHub] [druid] jihoonson closed pull request #11643: Cancel API for sqls

Posted by GitBox <gi...@apache.org>.
jihoonson closed pull request #11643:
URL: https://github.com/apache/druid/pull/11643


   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@druid.apache.org
For additional commands, e-mail: commits-help@druid.apache.org