Yingyi Bu has uploaded a new change for review.
https://asterix-gerrit.ics.uci.edu/1564
Change subject: Add a REST endpoint for query cancellation.
......................................................................
Add a REST endpoint for query cancellation.
Change-Id: I2936ac83f71bbef533e2695ed0a2b220c23fc483
---
M
asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutor.java
A
asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutorContext.java
A
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/ctx/StatementExecutorContext.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/ApiServlet.java
A
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryCancellationServlet.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/RestApiServlet.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/servlet/ServletConstants.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/api/java/AsterixJavaClient.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/app/translator/QueryTranslator.java
M
asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/CCApplicationEntryPoint.java
A
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/CancellationTestExecutor.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/ResultExtractor.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/TestExecutor.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/APIExecutionTest.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionFullParallelismIT.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionLessParallelismIT.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionTest.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateDefaultParameterTest.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionFullParallelismTest.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionLessParallelismTest.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/LangExecutionUtil.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionFullParallelismIT.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionLessParallelismIT.java
M
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionTest.java
A
asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionWithCancellationTest.java
M
asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/types/any-object/any-object.2.query.sqlpp
M
asterixdb/asterix-app/src/test/resources/runtimets/results/types/any-object/any-object.2.adm
M
asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/DatasetLifecycleManager.java
M
asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/Info.java
M
asterixdb/asterix-common/src/main/java/org/apache/asterix/common/utils/Servlets.java
M
asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/operators/LSMPrimaryUpsertOperatorNodePushable.java
M
hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/FileHandle.java
M
hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/IOManager.java
M
hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/MaterializingPipelinedPartition.java
M
hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/PipelinedPartition.java
M
hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/RunFileReader.java
M
hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/connectors/PartitionDataWriter.java
M
hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
M
hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexDataflowHelper.java
M
hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexSearchOperatorNodePushable.java
M
hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/TreeIndexStatsOperatorNodePushable.java
M
hyracks-fullstack/hyracks/hyracks-storage-am-lsm-common/src/main/java/org/apache/hyracks/storage/am/lsm/common/dataflow/LSMIndexCompactOperatorNodePushable.java
43 files changed, 688 insertions(+), 258 deletions(-)
git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb
refs/changes/64/1564/1
diff --git
a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutor.java
b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutor.java
index 2066f73..8156be1 100644
---
a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutor.java
+++
b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutor.java
@@ -85,26 +85,12 @@
* A Hyracks dataset client object that is used to read the
results.
* @param resultDelivery
* The {@code ResultDelivery} kind required for queries in the
list of statements
- * @throws Exception
- */
- void compileAndExecute(IHyracksClientConnection hcc, IHyracksDataset hdc,
ResultDelivery resultDelivery)
- throws Exception;
-
- /**
- * Compiles and execute a list of statements.
- *
- * @param hcc
- * A Hyracks client connection that is used to submit a jobspec
to Hyracks.
- * @param hdc
- * A Hyracks dataset client object that is used to read the
results.
- * @param resultDelivery
- * The {@code ResultDelivery} kind required for queries in the
list of statements
* @param stats
* a reference to write the stats of executed queries
* @throws Exception
*/
void compileAndExecute(IHyracksClientConnection hcc, IHyracksDataset hdc,
ResultDelivery resultDelivery,
- Stats stats) throws Exception;
+ Stats stats, String clientContextId, IStatementExecutorContext
ctx) throws Exception;
/**
* rewrites and compiles query into a hyracks job specifications
diff --git
a/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutorContext.java
b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutorContext.java
new file mode 100644
index 0000000..9e4f564
--- /dev/null
+++
b/asterixdb/asterix-algebra/src/main/java/org/apache/asterix/translator/IStatementExecutorContext.java
@@ -0,0 +1,31 @@
+/*
+ * 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.asterix.translator;
+
+import org.apache.hyracks.api.job.JobId;
+
+public interface IStatementExecutorContext {
+
+ JobId get(String clientContextId);
+
+ void put(String clientContextId, JobId jobId);
+
+ void remove(String clientContextId);
+}
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/ctx/StatementExecutorContext.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/ctx/StatementExecutorContext.java
new file mode 100644
index 0000000..fd4d1d8
--- /dev/null
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/ctx/StatementExecutorContext.java
@@ -0,0 +1,46 @@
+/*
+ * 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.asterix.api.http.ctx;
+
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+
+import org.apache.asterix.translator.IStatementExecutorContext;
+import org.apache.hyracks.api.job.JobId;
+
+public class StatementExecutorContext implements IStatementExecutorContext {
+
+ private final Map<String, JobId> activeQueries = new ConcurrentHashMap<>();
+
+ @Override
+ public JobId get(String clientContextId) {
+ return activeQueries.get(clientContextId);
+ }
+
+ @Override
+ public void put(String clientContextId, JobId jobId) {
+ activeQueries.put(clientContextId, jobId);
+ }
+
+ @Override
+ public void remove(String clientContextId) {
+ activeQueries.remove(clientContextId);
+ }
+}
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/ApiServlet.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/ApiServlet.java
index 7a33c0e..5ad0943 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/ApiServlet.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/ApiServlet.java
@@ -145,7 +145,8 @@
compilationProvider, componentProvider);
double duration;
long startTime = System.currentTimeMillis();
- translator.compileAndExecute(hcc, hds,
IStatementExecutor.ResultDelivery.IMMEDIATE);
+ translator.compileAndExecute(hcc, hds,
IStatementExecutor.ResultDelivery.IMMEDIATE,
+ new IStatementExecutor.Stats(), null, null);
long endTime = System.currentTimeMillis();
duration = (endTime - startTime) / 1000.00;
out.println(HTML_STATEMENT_SEPARATOR);
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryCancellationServlet.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryCancellationServlet.java
new file mode 100644
index 0000000..e7f9185
--- /dev/null
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryCancellationServlet.java
@@ -0,0 +1,103 @@
+/*
+ * 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.asterix.api.http.server;
+
+import java.io.IOException;
+import java.io.PrintWriter;
+import java.util.concurrent.ConcurrentMap;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.asterix.api.http.servlet.ServletConstants;
+import org.apache.asterix.translator.IStatementExecutorContext;
+import org.apache.hyracks.api.client.IHyracksClientConnection;
+import org.apache.hyracks.api.job.JobId;
+import org.apache.hyracks.http.api.IServletRequest;
+import org.apache.hyracks.http.api.IServletResponse;
+import org.apache.hyracks.http.server.AbstractServlet;
+import org.apache.hyracks.http.server.utils.HttpUtil;
+
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.fasterxml.jackson.databind.SerializationFeature;
+import com.fasterxml.jackson.databind.node.ObjectNode;
+
+import io.netty.handler.codec.http.HttpResponseStatus;
+
+public class QueryCancellationServlet extends AbstractServlet {
+ private static final Logger LOGGER =
Logger.getLogger(QueryCancellationServlet.class.getName());
+ private static final String CLIENT_CONTEXT_ID = "client_context_id";
+ private static final String JOB_ID = "jobId";
+ private static final String CANCELLED = "cancelled";
+
+ public QueryCancellationServlet(ConcurrentMap<String, Object> ctx,
String[] paths) {
+ super(ctx, paths);
+ }
+
+ @Override
+ protected void post(IServletRequest request, IServletResponse response)
throws IOException {
+ // Parses clientContextId from the request uri.
+ String requestURI = request.getHttpRequest().uri().toString();
+ String clientContextId =
requestURI.substring(requestURI.lastIndexOf('/') + 1);
+
+ // Retrieves the corresponding Hyracks job id.
+ IStatementExecutorContext activeQueries = (IStatementExecutorContext)
ctx
+ .get(ServletConstants.ACTIVE_QUERIES_ATTR);
+ IHyracksClientConnection hcc = (IHyracksClientConnection)
ctx.get(ServletConstants.HYRACKS_CONNECTION_ATTR);
+ JobId jobId = activeQueries.get(clientContextId);
+
+ // Creates the response data.
+ ObjectMapper om = new ObjectMapper();
+ om.enable(SerializationFeature.INDENT_OUTPUT);
+ ObjectNode result = om.createObjectNode();
+ result.put(CLIENT_CONTEXT_ID, clientContextId);
+
+ // Gets the response writer.
+ response.setStatus(HttpResponseStatus.OK);
+ HttpUtil.setContentType(response,
HttpUtil.ContentType.APPLICATION_JSON, HttpUtil.Encoding.UTF8);
+ PrintWriter responseWriter = response.writer();
+ if (jobId == null) {
+ // Writes the result.
+ result.put(JOB_ID, "null");
+
responseWriter.write(om.writerWithDefaultPrettyPrinter().writeValueAsString(result));
+ return;
+ }
+
+ try {
+ // Cancels the ongoing job.
+ hcc.cancelJob(jobId);
+
+ // Adds response fields for the case of success.
+ result.put(JOB_ID, jobId.getId());
+ result.put(CANCELLED, true);
+
+ // Removes the cancelled query.
+ activeQueries.remove(clientContextId);
+ } catch (Exception e) {
+ if (LOGGER.isLoggable(Level.WARNING)) {
+ LOGGER.log(Level.WARNING, e.getMessage(), e);
+ }
+ // Adds response fields for the case of failure.
+ result.put(CANCELLED, false);
+ response.setStatus(HttpResponseStatus.INTERNAL_SERVER_ERROR);
+ } finally {
+ // Writes the result.
+
responseWriter.write(om.writerWithDefaultPrettyPrinter().writeValueAsString(result));
+ }
+ }
+}
\ No newline at end of file
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
index 9d22452..fd82e41 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/QueryServiceServlet.java
@@ -29,6 +29,8 @@
import java.util.logging.Level;
import java.util.logging.Logger;
+import org.apache.asterix.api.http.ctx.StatementExecutorContext;
+import org.apache.asterix.api.http.servlet.ServletConstants;
import org.apache.asterix.app.result.ResultUtil;
import org.apache.asterix.common.api.IClusterManagementWork;
import org.apache.asterix.common.config.GlobalConfig;
@@ -43,6 +45,7 @@
import org.apache.asterix.runtime.utils.ClusterStateManager;
import org.apache.asterix.translator.IStatementExecutor;
import org.apache.asterix.translator.IStatementExecutor.Stats;
+import org.apache.asterix.translator.IStatementExecutorContext;
import org.apache.asterix.translator.IStatementExecutorFactory;
import org.apache.asterix.translator.SessionConfig;
import org.apache.hyracks.algebricks.common.exceptions.AlgebricksException;
@@ -65,6 +68,7 @@
private final ILangCompilationProvider compilationProvider;
private final IStatementExecutorFactory statementExecutorFactory;
private final IStorageComponentProvider componentProvider;
+ private final IStatementExecutorContext queryCtx = new
StatementExecutorContext();
public QueryServiceServlet(ConcurrentMap<String, Object> ctx, String[]
paths,
ILangCompilationProvider compilationProvider,
IStatementExecutorFactory statementExecutorFactory,
@@ -73,6 +77,7 @@
this.compilationProvider = compilationProvider;
this.statementExecutorFactory = statementExecutorFactory;
this.componentProvider = componentProvider;
+ ctx.put(ServletConstants.ACTIVE_QUERIES_ATTR, queryCtx);
}
@Override
@@ -425,7 +430,8 @@
IStatementExecutor translator =
statementExecutorFactory.create(statements, sessionConfig,
compilationProvider, componentProvider);
execStart = System.nanoTime();
- translator.compileAndExecute(getHyracksClientConnection(),
getHyracksDataset(), delivery, stats);
+ translator.compileAndExecute(getHyracksClientConnection(),
getHyracksDataset(), delivery, stats,
+ param.clientContextID, queryCtx);
execEnd = System.nanoTime();
printStatus(resultWriter, ResultDelivery.ASYNC == delivery ?
ResultStatus.RUNNING : ResultStatus.SUCCESS);
} catch (AsterixException | TokenMgrError |
org.apache.asterix.aqlplus.parser.TokenMgrError pe) {
@@ -459,4 +465,4 @@
LOGGER.warning("Error flushing output writer");
}
}
-}
+}
\ No newline at end of file
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/RestApiServlet.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/RestApiServlet.java
index bdc9d62..d32f51c 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/RestApiServlet.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/server/RestApiServlet.java
@@ -203,7 +203,7 @@
MetadataManager.INSTANCE.init();
IStatementExecutor translator =
statementExecutorFactory.create(aqlStatements, sessionConfig,
compilationProvider, componentProvider);
- translator.compileAndExecute(hcc, hds, resultDelivery);
+ translator.compileAndExecute(hcc, hds, resultDelivery, new
IStatementExecutor.Stats(), null, null);
} catch (AsterixException | TokenMgrError |
org.apache.asterix.aqlplus.parser.TokenMgrError pe) {
response.setStatus(HttpResponseStatus.INTERNAL_SERVER_ERROR);
GlobalConfig.ASTERIX_LOGGER.log(Level.SEVERE, pe.getMessage(), pe);
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/servlet/ServletConstants.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/servlet/ServletConstants.java
index 5b96cab..f5f6572 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/servlet/ServletConstants.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/http/servlet/ServletConstants.java
@@ -23,6 +23,7 @@
public static final String HYRACKS_DATASET_ATTR =
"org.apache.asterix.HYRACKS_DATASET";
public static final String ASTERIX_APP_CONTEXT_INFO_ATTR =
"org.apache.asterix.APP_CONTEXT_INFO";
public static final String EXECUTOR_SERVICE =
"org.apache.asterix.EXECUTOR_SERVICE";
+ public static final String ACTIVE_QUERIES_ATTR =
"org.apache.asterix.ACTIVE_QUERIES";
private ServletConstants() {
}
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/java/AsterixJavaClient.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/java/AsterixJavaClient.java
index c09f8cb..7ac3cfb 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/java/AsterixJavaClient.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/java/AsterixJavaClient.java
@@ -104,7 +104,8 @@
IStatementExecutor translator =
statementExecutorFactory.create(statements, conf,
compilationProvider, storageComponentProvider);
- translator.compileAndExecute(hcc, null,
QueryTranslator.ResultDelivery.IMMEDIATE);
+ translator.compileAndExecute(hcc, null,
QueryTranslator.ResultDelivery.IMMEDIATE,
+ new IStatementExecutor.Stats(), null, null);
writer.flush();
}
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/translator/QueryTranslator.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/translator/QueryTranslator.java
index 1e4d866..92119a1 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/translator/QueryTranslator.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/translator/QueryTranslator.java
@@ -156,6 +156,7 @@
import
org.apache.asterix.translator.CompiledStatements.CompiledUpsertStatement;
import org.apache.asterix.translator.CompiledStatements.ICompiledDmlStatement;
import org.apache.asterix.translator.IStatementExecutor;
+import org.apache.asterix.translator.IStatementExecutorContext;
import org.apache.asterix.translator.SessionConfig;
import org.apache.asterix.translator.TypeTranslator;
import org.apache.asterix.translator.util.ValidateUtil;
@@ -232,27 +233,9 @@
return functionDecls;
}
- /**
- * Compiles and submits for execution a list of AQL statements.
- *
- * @param hcc
- * A Hyracks client connection that is used to submit a jobspec
to Hyracks.
- * @param hdc
- * A Hyracks dataset client object that is used to read the
results.
- * @param resultDelivery
- * True if the results should be read asynchronously or false
if we should wait for results to be read.
- * @return A List<QueryResult> containing a QueryResult instance
corresponding to each submitted query.
- * @throws Exception
- */
- @Override
- public void compileAndExecute(IHyracksClientConnection hcc,
IHyracksDataset hdc, ResultDelivery resultDelivery)
- throws Exception {
- compileAndExecute(hcc, hdc, resultDelivery, new Stats());
- }
-
@Override
public void compileAndExecute(IHyracksClientConnection hcc,
IHyracksDataset hdc, ResultDelivery resultDelivery,
- Stats stats) throws Exception {
+ Stats stats, String clientContextId, IStatementExecutorContext
ctx) throws Exception {
int resultSetIdCounter = 0;
FileSplit outputFile = null;
IAWriterFactory writerFactory = PrinterBasedWriterFactory.INSTANCE;
@@ -329,7 +312,8 @@
metadataProvider.setResultAsyncMode(resultDelivery
== ResultDelivery.ASYNC
|| resultDelivery ==
ResultDelivery.DEFERRED);
}
- handleInsertUpsertStatement(metadataProvider, stmt,
hcc, hdc, resultDelivery, stats, false);
+ handleInsertUpsertStatement(metadataProvider, stmt,
hcc, hdc, resultDelivery, stats, false,
+ clientContextId, ctx);
break;
case Statement.Kind.DELETE:
handleDeleteStatement(metadataProvider, stmt, hcc,
false);
@@ -362,7 +346,8 @@
metadataProvider.setResultSetId(new
ResultSetId(resultSetIdCounter++));
metadataProvider.setResultAsyncMode(
resultDelivery == ResultDelivery.ASYNC ||
resultDelivery == ResultDelivery.DEFERRED);
- handleQuery(metadataProvider, (Query) stmt, hcc, hdc,
resultDelivery, stats);
+ handleQuery(metadataProvider, (Query) stmt, hcc, hdc,
resultDelivery, stats, clientContextId,
+ ctx);
break;
case Statement.Kind.COMPACT:
handleCompactStatement(metadataProvider, stmt, hcc);
@@ -1809,8 +1794,8 @@
public JobSpecification handleInsertUpsertStatement(MetadataProvider
metadataProvider, Statement stmt,
IHyracksClientConnection hcc, IHyracksDataset hdc, ResultDelivery
resultDelivery,
- IStatementExecutor.Stats stats, boolean compileOnly) throws
Exception {
-
+ IStatementExecutor.Stats stats, boolean compileOnly, String
clientContextId, IStatementExecutorContext ctx)
+ throws Exception {
InsertStatement stmtInsertUpsert = (InsertStatement) stmt;
String dataverseName =
getActiveDataverse(stmtInsertUpsert.getDataverseName());
Query query = stmtInsertUpsert.getQuery();
@@ -1852,7 +1837,7 @@
}
if (stmtInsertUpsert.getReturnExpression() != null) {
- deliverResult(hcc, hdc, compiler, metadataProvider, locker,
resultDelivery, stats);
+ deliverResult(hcc, hdc, compiler, metadataProvider, locker,
resultDelivery, stats, clientContextId, ctx);
} else {
locker.lock();
try {
@@ -2371,7 +2356,8 @@
}
protected void handleQuery(MetadataProvider metadataProvider, Query query,
IHyracksClientConnection hcc,
- IHyracksDataset hdc, ResultDelivery resultDelivery, Stats stats)
throws Exception {
+ IHyracksDataset hdc, ResultDelivery resultDelivery, Stats stats,
String clientContextId,
+ IStatementExecutorContext ctx) throws Exception {
final IMetadataLocker locker = new IMetadataLocker() {
@Override
public void lock() {
@@ -2402,11 +2388,12 @@
throw e;
}
};
- deliverResult(hcc, hdc, compiler, metadataProvider, locker,
resultDelivery, stats);
+ deliverResult(hcc, hdc, compiler, metadataProvider, locker,
resultDelivery, stats, clientContextId, ctx);
}
private void deliverResult(IHyracksClientConnection hcc, IHyracksDataset
hdc, IStatementCompiler compiler,
- MetadataProvider metadataProvider, IMetadataLocker locker,
ResultDelivery resultDelivery, Stats stats)
+ MetadataProvider metadataProvider, IMetadataLocker locker,
ResultDelivery resultDelivery, Stats stats,
+ String clientContextId, IStatementExecutorContext ctx)
throws Exception {
final ResultSetId resultSetId = metadataProvider.getResultSetId();
switch (resultDelivery) {
@@ -2422,7 +2409,7 @@
printed.setTrue();
printed.notify();
}
- });
+ }, clientContextId, ctx);
} catch (Exception e) {
GlobalConfig.ASTERIX_LOGGER.log(Level.SEVERE,
resultDelivery.name() + " job " + "with id " +
jobId + " failed", e);
@@ -2439,12 +2426,12 @@
final ResultReader resultReader = new ResultReader(hdc,
id, resultSetId);
ResultUtil.printResults(resultReader, sessionConfig, stats,
metadataProvider.findOutputRecordType());
- });
+ }, clientContextId, ctx);
break;
case DEFERRED:
createAndRunJob(hcc, compiler, locker, resultDelivery, id -> {
ResultUtil.printResultHandle(new ResultHandle(id,
resultSetId), sessionConfig);
- });
+ }, clientContextId, ctx);
break;
default:
break;
@@ -2452,7 +2439,8 @@
}
private static JobId createAndRunJob(IHyracksClientConnection hcc,
IStatementCompiler compiler,
- IMetadataLocker locker, ResultDelivery resultDelivery,
IResultPrinter printer) throws Exception {
+ IMetadataLocker locker, ResultDelivery resultDelivery,
IResultPrinter printer, String clientContextId,
+ IStatementExecutorContext ctx) throws Exception {
locker.lock();
try {
final JobSpecification jobSpec = compiler.compile();
@@ -2460,6 +2448,10 @@
return JobId.INVALID;
}
final JobId jobId = JobUtils.runJob(hcc, jobSpec, false);
+
+ if (ctx != null && clientContextId != null) {
+ ctx.put(clientContextId, jobId); // Adds the running job into
the context.
+ }
if (ResultDelivery.ASYNC == resultDelivery) {
printer.print(jobId);
hcc.waitForCompletion(jobId);
@@ -2467,6 +2459,9 @@
hcc.waitForCompletion(jobId);
printer.print(jobId);
}
+ if (ctx != null && clientContextId != null) {
+ ctx.remove(clientContextId); // Removes the running job into
the context.
+ }
return jobId;
} finally {
locker.unlock();
diff --git
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/CCApplicationEntryPoint.java
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/CCApplicationEntryPoint.java
index b63bb31..7629908 100644
---
a/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/CCApplicationEntryPoint.java
+++
b/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/CCApplicationEntryPoint.java
@@ -16,6 +16,7 @@
* specific language governing permissions and limitations
* under the License.
*/
+
package org.apache.asterix.hyracks.bootstrap;
import static
org.apache.asterix.api.http.servlet.ServletConstants.ASTERIX_APP_CONTEXT_INFO_ATTR;
@@ -38,6 +39,7 @@
import org.apache.asterix.api.http.server.FullApiServlet;
import org.apache.asterix.api.http.server.NodeControllerDetailsApiServlet;
import org.apache.asterix.api.http.server.QueryApiServlet;
+import org.apache.asterix.api.http.server.QueryCancellationServlet;
import org.apache.asterix.api.http.server.QueryResultApiServlet;
import org.apache.asterix.api.http.server.QueryServiceServlet;
import org.apache.asterix.api.http.server.QueryStatusApiServlet;
@@ -72,6 +74,7 @@
import org.apache.asterix.runtime.utils.AppContextInfo;
import org.apache.asterix.translator.IStatementExecutorFactory;
import org.apache.hyracks.api.application.ICCApplicationContext;
+import org.apache.hyracks.api.application.ICCApplicationEntryPoint;
import org.apache.hyracks.api.client.HyracksConnection;
import org.apache.hyracks.api.client.IHyracksClientConnection;
import org.apache.hyracks.api.config.IConfigManager;
@@ -206,6 +209,7 @@
addServlet(jsonAPIServer, Servlets.QUERY_STATUS);
addServlet(jsonAPIServer, Servlets.QUERY_RESULT);
addServlet(jsonAPIServer, Servlets.QUERY_SERVICE);
+ addServlet(jsonAPIServer, Servlets.QUERY_CANCEL);
addServlet(jsonAPIServer, Servlets.CONNECTOR);
addServlet(jsonAPIServer, Servlets.SHUTDOWN);
addServlet(jsonAPIServer, Servlets.VERSION);
@@ -263,6 +267,8 @@
case Servlets.SQLPP_DDL:
return new DdlApiServlet(ctx, paths,
ccExtensionManager.getSqlppCompilationProvider(),
getStatementExecutorFactory(), componentProvider);
+ case Servlets.QUERY_CANCEL:
+ return new QueryCancellationServlet(ctx, paths);
case Servlets.QUERY_STATUS:
return new QueryStatusApiServlet(ctx, paths);
case Servlets.QUERY_RESULT:
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/CancellationTestExecutor.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/CancellationTestExecutor.java
new file mode 100644
index 0000000..db072a3
--- /dev/null
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/CancellationTestExecutor.java
@@ -0,0 +1,72 @@
+/*
+ * 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.asterix.test.common;
+
+import java.io.InputStream;
+import java.net.URI;
+import java.util.List;
+import java.util.UUID;
+import java.util.concurrent.Callable;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.Future;
+
+import org.apache.asterix.common.utils.Servlets;
+import org.apache.asterix.testframework.context.TestCaseContext;
+import org.apache.asterix.testframework.xml.TestCase;
+import org.apache.http.HttpResponse;
+import org.apache.http.client.methods.HttpUriRequest;
+
+public class CancellationTestExecutor extends TestExecutor {
+
+ private ExecutorService executor = Executors.newSingleThreadExecutor();
+
+ @Override
+ public InputStream executeQueryService(String str,
TestCaseContext.OutputFormat fmt, URI uri,
+ List<TestCase.CompilationUnit.Parameter> params, boolean
jsonEncoded, boolean cancellable)
+ throws Exception {
+ String clientContextId = UUID.randomUUID().toString();
+ if (cancellable) {
+ setParam(params, "client_context_id", clientContextId);
+ }
+ Callable<InputStream> query = () -> {
+ try {
+ return CancellationTestExecutor.super.executeQueryService(str,
fmt, uri, params, jsonEncoded, true);
+ } catch (Exception e) {
+ e.printStackTrace();
+ throw e;
+ }
+ };
+ Future<InputStream> future = executor.submit(query);
+ if (cancellable) {
+ Thread.sleep(20);
+ // Cancels the query request while the query is executing.
+ cancelQuery(str, getEndpoint(Servlets.QUERY_CANCEL),
clientContextId);
+ }
+ return future.get();
+ }
+
+ private InputStream cancelQuery(String str, URI uri, String
clientContextId) throws Exception {
+ URI baseURI = new URI(uri.toString() + "/");
+ HttpUriRequest method = constructPostMethodUrl(str,
baseURI.resolve(clientContextId), null, null);
+ HttpResponse response = executeHttpRequest(method);
+ return response.getEntity().getContent();
+ }
+}
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/ResultExtractor.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/ResultExtractor.java
index 3531211..53a0f6c 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/ResultExtractor.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/ResultExtractor.java
@@ -19,7 +19,6 @@
package org.apache.asterix.test.common;
import java.io.InputStream;
-import java.io.StringWriter;
import java.nio.charset.Charset;
import java.util.Iterator;
import java.util.logging.Logger;
@@ -59,6 +58,8 @@
field = sIter.next();
switch (field) {
case "requestID":
+ break;
+ case "clientContextID":
break;
case "signature":
break;
@@ -106,7 +107,7 @@
}
break;
default:
- throw new AsterixException(field + "unanticipated field");
+ throw new AsterixException(field + " unanticipated field");
}
}
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/TestExecutor.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/TestExecutor.java
index 7765572..a1fcd3f 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/TestExecutor.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/common/TestExecutor.java
@@ -457,6 +457,11 @@
public InputStream executeQueryService(String str, OutputFormat fmt, URI
uri,
List<CompilationUnit.Parameter> params, boolean jsonEncoded)
throws Exception {
+ return executeQueryService(str, fmt, uri, params, jsonEncoded, false);
+ }
+
+ public InputStream executeQueryService(String str, OutputFormat fmt, URI
uri,
+ List<CompilationUnit.Parameter> params, boolean jsonEncoded,
boolean cancellable) throws Exception {
setParam(params, "format", fmt.mimeType());
HttpUriRequest method = jsonEncoded ? constructPostMethodJson(str,
uri, "statement", params)
: constructPostMethodUrl(str, uri, "statement", params);
@@ -830,12 +835,12 @@
}
final URI uri = getEndpoint(Servlets.QUERY_SERVICE);
if (DELIVERY_IMMEDIATE.equals(delivery)) {
- resultStream = executeQueryService(statement, fmt,
uri, params, true);
+ resultStream = executeQueryService(statement, fmt,
uri, params, true, true);
resultStream = ResultExtractor.extract(resultStream);
} else {
String handleVar = getHandleVariable(statement);
setParam(params, "mode", delivery);
- resultStream = executeQueryService(statement, fmt,
uri, params, true);
+ resultStream = executeQueryService(statement, fmt,
uri, params, true, true);
String handle =
ResultExtractor.extractHandle(resultStream);
Assert.assertNotNull("no handle for " + reqType + "
test " + testFile.toString(), handleVar);
variableCtx.put(handleVar, handle);
@@ -1221,6 +1226,9 @@
} catch (JsonMappingException e) {
result = om.createArrayNode();
}
+ if (result == null) {
+ return;
+ }
for (int i = 0; i < result.size(); i++) {
JsonNode json = result.get(i);
if (json != null) {
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/APIExecutionTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/APIExecutionTest.java
index 82f90ec..dd87455 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/APIExecutionTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/APIExecutionTest.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionFullParallelismIT.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionFullParallelismIT.java
index 8df93ba..6cc5a9c 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionFullParallelismIT.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionFullParallelismIT.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionLessParallelismIT.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionLessParallelismIT.java
index 2f3c395..dc03626 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionLessParallelismIT.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionLessParallelismIT.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionTest.java
index 4169a07..abc9f2f 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/AqlExecutionTest.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateDefaultParameterTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateDefaultParameterTest.java
index 86a9639..17e88a6 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateDefaultParameterTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateDefaultParameterTest.java
@@ -25,6 +25,7 @@
import java.net.URL;
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.Assert;
@@ -43,7 +44,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionFullParallelismTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionFullParallelismTest.java
index b7b4312..e428c93 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionFullParallelismTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionFullParallelismTest.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionLessParallelismTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionLessParallelismTest.java
index 9516d7d..346ae2f 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionLessParallelismTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/ClusterStateExecutionLessParallelismTest.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/LangExecutionUtil.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/LangExecutionUtil.java
index 0e6be0f..52e98f1 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/LangExecutionUtil.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/LangExecutionUtil.java
@@ -35,7 +35,6 @@
import org.apache.asterix.testframework.context.TestCaseContext;
import org.apache.commons.lang.SystemUtils;
import org.apache.commons.lang3.StringUtils;
-import org.junit.Assert;
import org.junit.runner.RunWith;
import org.junit.runners.Parameterized;
@@ -52,12 +51,13 @@
private static final boolean cleanupOnStart = true;
private static final boolean cleanupOnStop = true;
private static final List<String> badTestCases = new ArrayList<>();
- private static final TestExecutor testExecutor = new TestExecutor();
+ private static TestExecutor testExecutor;
private static TestLibrarian librarian;
private static final int repeat = Integer.getInteger("test.repeat", 1);
- public static void setUp(String configFile) throws Exception {
+ public static void setUp(String configFile, TestExecutor executor) throws
Exception {
+ testExecutor = executor;
File outdir = new File(PATH_ACTUAL);
outdir.mkdirs();
List<ILibraryManager> libraryManagers =
ExecutionTestUtil.setUp(cleanupOnStart, configFile);
@@ -142,7 +142,21 @@
.exec(new String[] { "bash", "-c", "lsof -p " + processId +
"|grep waf|wc -l" });
try (BufferedReader reader = new BufferedReader(new
InputStreamReader(process.getInputStream()))) {
int runFileCount = Integer.parseInt(reader.readLine().trim());
- Assert.assertTrue(runFileCount == 0);
+ if (runFileCount != 0) {
+ outputLeakedFiles(processId);
+ throw new AssertionError("There are " + runFileCount + "
leaked run files.");
+ }
+ }
+ }
+
+ private static void outputLeakedFiles(String processId) throws IOException
{
+ Process process = Runtime.getRuntime()
+ .exec(new String[] { "bash", "-c", "lsof -p " + processId +
"|grep waf" });
+ try (BufferedReader reader = new BufferedReader(new
InputStreamReader(process.getInputStream()))) {
+ String line;
+ while ((line = reader.readLine()) != null) {
+ System.err.println(line);
+ }
}
}
}
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionFullParallelismIT.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionFullParallelismIT.java
index f2372ed..3d93c36 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionFullParallelismIT.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionFullParallelismIT.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionLessParallelismIT.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionLessParallelismIT.java
index c3cc058..53b068e 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionLessParallelismIT.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionLessParallelismIT.java
@@ -21,6 +21,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -38,7 +39,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionTest.java
index f19ebbf..8ec1fe7 100644
---
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionTest.java
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionTest.java
@@ -20,6 +20,7 @@
import java.util.Collection;
+import org.apache.asterix.test.common.TestExecutor;
import org.apache.asterix.testframework.context.TestCaseContext;
import org.junit.AfterClass;
import org.junit.BeforeClass;
@@ -37,7 +38,7 @@
@BeforeClass
public static void setUp() throws Exception {
- LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME);
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new TestExecutor());
}
@AfterClass
diff --git
a/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionWithCancellationTest.java
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionWithCancellationTest.java
new file mode 100644
index 0000000..668f109
--- /dev/null
+++
b/asterixdb/asterix-app/src/test/java/org/apache/asterix/test/runtime/SqlppExecutionWithCancellationTest.java
@@ -0,0 +1,95 @@
+/*
+ * 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.asterix.test.runtime;
+
+import java.util.Collection;
+
+import org.apache.asterix.test.common.CancellationTestExecutor;
+import org.apache.asterix.testframework.context.TestCaseContext;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.Parameterized;
+import org.junit.runners.Parameterized.Parameters;
+
+/**
+ * Runs the SQL++ runtime tests with the storage parallelism.
+ */
+@RunWith(Parameterized.class)
+public class SqlppExecutionWithCancellationTest {
+ protected static final String TEST_CONFIG_FILE_NAME =
"asterix-build-configuration.xml";
+ private static int numCancelledQueries = 0;
+
+ @BeforeClass
+ public static void setUp() throws Exception {
+ LangExecutionUtil.setUp(TEST_CONFIG_FILE_NAME, new
CancellationTestExecutor());
+ }
+
+ @AfterClass
+ public static void tearDown() throws Exception {
+ System.err.println(numCancelledQueries + " queries has been cancelled
during the test.");
+ LangExecutionUtil.tearDown();
+ }
+
+ @Parameters(name = "SqlppExecutionWithCancellationTest {index}: {0}")
+ public static Collection<Object[]> tests() throws Exception {
+ return LangExecutionUtil.tests("only_sqlpp.xml",
"testsuite_sqlpp.xml");
+ }
+
+ protected TestCaseContext tcCtx;
+
+ public SqlppExecutionWithCancellationTest(TestCaseContext tcCtx) {
+ this.tcCtx = tcCtx;
+ }
+
+ @Test
+ public void test() throws Exception {
+ try {
+ LangExecutionUtil.test(tcCtx);
+ } catch (Exception e) {
+ Throwable cause = getRootCause(e);
+ String errorMsg = cause.getMessage();
+ // "HYR0025" means a user cancelled the query.
+ // "status" check result can change because of cancellation.
+ if (errorMsg.startsWith("HYR0025") // Expected
+ || errorMsg.contains("\"status\": ") // Expected
+ || errorMsg.contains("reference count") // Unexpected:
ASTERIXDB
+ || errorMsg.contains("Resource doesn't exist") //
Unexpected: ASTERIXDB
+ || errorMsg.contains("is pinned and file is being closed")
// Unexpected: ASTERIXDB
+ ) {
+ numCancelledQueries++;
+ } else {
+ // Re-throw other kinds of exceptions.
+ throw e;
+ }
+ }
+ }
+
+ private Throwable getRootCause(Throwable e) {
+ Throwable current = e;
+ Throwable cause = e.getCause();
+ while (cause != null) {
+ Throwable nextCause = current.getCause();
+ current = cause;
+ cause = nextCause;
+ }
+ return current;
+ }
+}
diff --git
a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/types/any-object/any-object.2.query.sqlpp
b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/types/any-object/any-object.2.query.sqlpp
index afaa5b1..81d65a9 100644
---
a/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/types/any-object/any-object.2.query.sqlpp
+++
b/asterixdb/asterix-app/src/test/resources/runtimets/queries_sqlpp/types/any-object/any-object.2.query.sqlpp
@@ -17,4 +17,6 @@
* under the License.
*/
-select element x from Metadata.Datatype as x;
+select element x from Metadata.Datatype as x
+where DataverseName = "test" or DataverseName = "Metadata"
+order by DataversName, DatatypeName;
diff --git
a/asterixdb/asterix-app/src/test/resources/runtimets/results/types/any-object/any-object.2.adm
b/asterixdb/asterix-app/src/test/resources/runtimets/results/types/any-object/any-object.2.adm
index 91df332..0e57bc0 100644
---
a/asterixdb/asterix-app/src/test/resources/runtimets/results/types/any-object/any-object.2.adm
+++
b/asterixdb/asterix-app/src/test/resources/runtimets/results/types/any-object/any-object.2.adm
@@ -1,69 +1,69 @@
-{ "DataverseName": "Metadata", "DatatypeName": "AnyObject", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "CompactionPolicyRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "CompactionPolicy", "FieldType":
"string", "IsNullable": false }, { "FieldName": "Classname", "FieldType":
"string", "IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT
2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType", "Derived":
{ "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields":
[ { "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false },
{ "FieldName": "DatasetName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatatypeDataverseName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "DatatypeName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "DatasetType", "FieldType": "string", "IsNullable":
false }, { "FieldName": "GroupName", "FieldType": "string", "IsNullable": false
}, { "FieldName": "CompactionPolicy", "FieldType": "string", "IsNullable":
false }, { "FieldName": "CompactionPolicyProperties", "FieldType":
"DatasetRecordType_CompactionPolicyProperties", "IsNullable": false }, {
"FieldName": "InternalDetails", "FieldType":
"DatasetRecordType_InternalDetails", "IsNullable": true }, { "FieldName":
"ExternalDetails", "FieldType":
"DatasetRecordType_ExternalDetails", "IsNullable": true }, { "FieldName":
"Hints", "FieldType": "DatasetRecordType_Hints", "IsNullable": false }, {
"FieldName": "Timestamp", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatasetId", "FieldType": "int32", "IsNullable": false }, {
"FieldName": "PendingOp", "FieldType": "int32", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_CompactionPolicyProperties", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_CompactionPolicyProperties_Item" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_CompactionPolicyProperties_Item", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"DatasourceAdapter", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Properties", "FieldType":
"DatasetRecordType_ExternalDetails_Properties", "IsNullable": false }, {
"FieldName": "LastRefreshTime", "FieldType": "datetime", "IsNullable": false },
{ "FieldName": "TransactionState", "FieldType": "int32", "IsNullable": false }
] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails_Properties", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_ExternalDetails_Properties_Item" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails_Properties_Item", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType_Hints",
"Derived": { "Tag": "UNORDEREDLIST", "IsAnonymous": true, "UnorderedList":
"DatasetRecordType_Hints_Item" }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType_Hints_Item",
"Derived": { "Tag": "RECORD", "IsAnonymous": true, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "Name", "FieldType": "string", "IsNullable": false
}, { "FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"FileStructure", "FieldType": "string", "IsNullable": false }, { "FieldName":
"PartitioningStrategy", "FieldType": "string", "IsNullable": false }, {
"FieldName": "PartitioningKey", "FieldType":
"DatasetRecordType_InternalDetails_PartitioningKey", "IsNullable": false }, {
"FieldName": "PrimaryKey", "FieldType":
"DatasetRecordType_InternalDetails_PrimaryKey", "IsNullable": false }, {
"FieldName": "Autogenerated", "FieldType": "boolean", "IsNullable": false } ] }
}, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PartitioningKey", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_InternalDetails_PartitioningKey_Item" }, "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PartitioningKey_Item", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string" }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PrimaryKey", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_InternalDetails_PrimaryKey_Item" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PrimaryKey_Item", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string" }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatasourceAdapterRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Name", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Classname", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Type", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatatypeRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatatypeName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Derived", "FieldType":
"DatatypeRecordType_Derived", "IsNullable": true }, { "FieldName": "Timestamp",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DatatypeRecordType_Derived",
"Derived": { "Tag": "RECORD", "IsAnonymous": true, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "Tag", "FieldType": "string", "IsNullable": false },
{ "FieldName": "IsAnonymous", "FieldType": "boolean", "IsNullable": false }, {
"FieldName": "Record", "FieldType": "DatatypeRecordType_Derived_Record",
"IsNullable": true }, { "FieldName": "UnorderedList", "FieldType": "string",
"IsNullable": true }, { "FieldName": "OrderedList", "FieldType": "string",
"IsNullable": true } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"IsOpen", "FieldType": "boolean", "IsNullable": false }, { "FieldName":
"Fields", "FieldType": "DatatypeRecordType_Derived_Record_Fields",
"IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record_Fields", "Derived": { "Tag": "ORDEREDLIST",
"IsAnonymous": true, "OrderedList":
"DatatypeRecordType_Derived_Record_Fields_Item" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record_Fields_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"FieldName", "FieldType": "string", "IsNullable": false }, { "FieldName":
"FieldType", "FieldType": "string", "IsNullable": false }, { "FieldName":
"IsNullable", "FieldType": "boolean", "IsNullable": false } ] } }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "DataverseRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DataFormat", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "ExternalFileRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatasetName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FileNumber", "FieldType": "int32",
"IsNullable": false }, { "FieldName": "FileName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FileSize", "FieldType": "int64",
"IsNullable": false }, { "FieldName": "FileModTime", "FieldType": "datetime",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "FeedConnectionRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FeedName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatasetName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "ReturnType", "FieldType": "string",
"IsNullable": false }, { "FieldName": "AppliedFunctions", "FieldType":
"FeedConnectionRecordType_AppliedFunctions", "IsNullable": false }, {
"FieldName": "PolicyName", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"FeedConnectionRecordType_AppliedFunctions", "Derived": { "Tag":
"UNORDEREDLIST", "IsAnonymous": true, "UnorderedList": "string" }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "FeedPolicyRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PolicyName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Description", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Properties", "FieldType":
"FeedPolicyRecordType_Properties", "IsNullable": false } ] } }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"FeedPolicyRecordType_Properties", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList": "FeedPolicyRecordType_Properties_Item" },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"FeedPolicyRecordType_Properties_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"Name", "FieldType": "string", "IsNullable": false }, { "FieldName": "Value",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "FeedRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "FeedName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "AdapterName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "AdapterConfiguration", "FieldType":
"FeedRecordType_AdapterConfiguration", "IsNullable": false }, { "FieldName":
"Timestamp", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"FeedRecordType_AdapterConfiguration", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList":
"FeedRecordType_AdapterConfiguration_Item" }, "Timestamp": "Mon Oct 10 14:53:55
PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"FeedRecordType_AdapterConfiguration_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"Name", "FieldType": "string", "IsNullable": false }, { "FieldName": "Value",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "FunctionRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Name", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Arity", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Params", "FieldType":
"FunctionRecordType_Params", "IsNullable": false }, { "FieldName":
"ReturnType", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Definition", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Language", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Kind", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "FunctionRecordType_Params",
"Derived": { "Tag": "ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string"
}, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "IndexRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatasetName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "IndexName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "IndexStructure", "FieldType": "string", "IsNullable": false }, {
"FieldName": "SearchKey", "FieldType": "IndexRecordType_SearchKey",
"IsNullable": false }, { "FieldName": "IsPrimary", "FieldType": "boolean",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "IndexRecordType_SearchKey",
"Derived": { "Tag": "ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"IndexRecordType_SearchKey_Item" }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016"
}
-{ "DataverseName": "Metadata", "DatatypeName":
"IndexRecordType_SearchKey_Item", "Derived": { "Tag": "ORDEREDLIST",
"IsAnonymous": true, "OrderedList": "string" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "LibraryRecordType", "Derived":
{ "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields":
[ { "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false },
{ "FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Timestamp", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "NodeGroupRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "GroupName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "NodeNames", "FieldType":
"NodeGroupRecordType_NodeNames", "IsNullable": false }, { "FieldName":
"Timestamp", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp":
"Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName":
"NodeGroupRecordType_NodeNames", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList": "string" }, "Timestamp": "Mon Oct 10
14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "NodeRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "NodeName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "NumberOfCores", "FieldType": "int64", "IsNullable": false }, {
"FieldName": "WorkingMemorySize", "FieldType": "int64", "IsNullable": false } ]
} }, "Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "binary", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "boolean", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "circle", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "date", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "datetime", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "day-time-duration",
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "double", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "duration", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "float", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "int16", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "int32", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "int64", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "int8", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "interval", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "line", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "missing", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "null", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "point", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "point3d", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "polygon", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "rectangle", "Timestamp": "Mon
Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "shortwithouttypeinfo",
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "string", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "time", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "uuid", "Timestamp": "Mon Oct
10 14:53:55 PDT 2016" }
-{ "DataverseName": "Metadata", "DatatypeName": "year-month-duration",
"Timestamp": "Mon Oct 10 14:53:55 PDT 2016" }
-{ "DataverseName": "test", "DatatypeName": "AnyObject", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ ] } },
"Timestamp": "Mon Oct 10 15:00:08 PDT 2016" }
-{ "DataverseName": "test", "DatatypeName": "kv1", "Derived": { "Tag":
"RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "id", "FieldType": "int32", "IsNullable": false }, { "FieldName":
"val", "FieldType": "AnyObject", "IsNullable": false } ] } }, "Timestamp": "Mon
Oct 10 15:00:08 PDT 2016" }
-{ "DataverseName": "test", "DatatypeName": "kv2", "Derived": { "Tag":
"RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "id", "FieldType": "int32", "IsNullable": false }, { "FieldName":
"val", "FieldType": "AnyObject", "IsNullable": false } ] } }, "Timestamp": "Mon
Oct 10 15:00:08 PDT 2016" }
+{ "DataverseName": "test", "DatatypeName": "AnyObject", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ ] } },
"Timestamp": "Thu Mar 09 15:46:46 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "AnyObject", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "CompactionPolicyRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "CompactionPolicy", "FieldType":
"string", "IsNullable": false }, { "FieldName": "Classname", "FieldType":
"string", "IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST
2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType", "Derived":
{ "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields":
[ { "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false },
{ "FieldName": "DatasetName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatatypeDataverseName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "DatatypeName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "DatasetType", "FieldType": "string", "IsNullable":
false }, { "FieldName": "GroupName", "FieldType": "string", "IsNullable": false
}, { "FieldName": "CompactionPolicy", "FieldType": "string", "IsNullable":
false }, { "FieldName": "CompactionPolicyProperties", "FieldType":
"DatasetRecordType_CompactionPolicyProperties", "IsNullable": false }, {
"FieldName": "InternalDetails", "FieldType":
"DatasetRecordType_InternalDetails", "IsNullable": true }, { "FieldName":
"ExternalDetails", "FieldType":
"DatasetRecordType_ExternalDetails", "IsNullable": true }, { "FieldName":
"Hints", "FieldType": "DatasetRecordType_Hints", "IsNullable": false }, {
"FieldName": "Timestamp", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatasetId", "FieldType": "int32", "IsNullable": false }, {
"FieldName": "PendingOp", "FieldType": "int32", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_CompactionPolicyProperties", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_CompactionPolicyProperties_Item" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_CompactionPolicyProperties_Item", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"DatasourceAdapter", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Properties", "FieldType":
"DatasetRecordType_ExternalDetails_Properties", "IsNullable": false }, {
"FieldName": "LastRefreshTime", "FieldType": "datetime", "IsNullable": false },
{ "FieldName": "TransactionState", "FieldType": "int32", "IsNullable": false }
] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails_Properties", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_ExternalDetails_Properties_Item" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_ExternalDetails_Properties_Item", "Derived": { "Tag":
"RECORD", "IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType_Hints",
"Derived": { "Tag": "UNORDEREDLIST", "IsAnonymous": true, "UnorderedList":
"DatasetRecordType_Hints_Item" }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatasetRecordType_Hints_Item",
"Derived": { "Tag": "RECORD", "IsAnonymous": true, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "Name", "FieldType": "string", "IsNullable": false
}, { "FieldName": "Value", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"FileStructure", "FieldType": "string", "IsNullable": false }, { "FieldName":
"PartitioningStrategy", "FieldType": "string", "IsNullable": false }, {
"FieldName": "PartitioningKey", "FieldType":
"DatasetRecordType_InternalDetails_PartitioningKey", "IsNullable": false }, {
"FieldName": "PrimaryKey", "FieldType":
"DatasetRecordType_InternalDetails_PrimaryKey", "IsNullable": false }, {
"FieldName": "Autogenerated", "FieldType": "boolean", "IsNullable": false } ] }
}, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PartitioningKey", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_InternalDetails_PartitioningKey_Item" }, "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PartitioningKey_Item", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string" }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PrimaryKey", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"DatasetRecordType_InternalDetails_PrimaryKey_Item" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatasetRecordType_InternalDetails_PrimaryKey_Item", "Derived": { "Tag":
"ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string" }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatasourceAdapterRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Name", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Classname", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Type", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatatypeRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatatypeName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Derived", "FieldType":
"DatatypeRecordType_Derived", "IsNullable": true }, { "FieldName": "Timestamp",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DatatypeRecordType_Derived",
"Derived": { "Tag": "RECORD", "IsAnonymous": true, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "Tag", "FieldType": "string", "IsNullable": false },
{ "FieldName": "IsAnonymous", "FieldType": "boolean", "IsNullable": false }, {
"FieldName": "Record", "FieldType": "DatatypeRecordType_Derived_Record",
"IsNullable": true }, { "FieldName": "UnorderedList", "FieldType": "string",
"IsNullable": true }, { "FieldName": "OrderedList", "FieldType": "string",
"IsNullable": true } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"IsOpen", "FieldType": "boolean", "IsNullable": false }, { "FieldName":
"Fields", "FieldType": "DatatypeRecordType_Derived_Record_Fields",
"IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record_Fields", "Derived": { "Tag": "ORDEREDLIST",
"IsAnonymous": true, "OrderedList":
"DatatypeRecordType_Derived_Record_Fields_Item" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"DatatypeRecordType_Derived_Record_Fields_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"FieldName", "FieldType": "string", "IsNullable": false }, { "FieldName":
"FieldType", "FieldType": "string", "IsNullable": false }, { "FieldName":
"IsNullable", "FieldType": "boolean", "IsNullable": false } ] } }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "DataverseRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DataFormat", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "ExternalFileRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatasetName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FileNumber", "FieldType": "int32",
"IsNullable": false }, { "FieldName": "FileName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FileSize", "FieldType": "int64",
"IsNullable": false }, { "FieldName": "FileModTime", "FieldType": "datetime",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "FeedConnectionRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "FeedName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "DatasetName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "ReturnType", "FieldType": "string",
"IsNullable": false }, { "FieldName": "AppliedFunctions", "FieldType":
"FeedConnectionRecordType_AppliedFunctions", "IsNullable": false }, {
"FieldName": "PolicyName", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"FeedConnectionRecordType_AppliedFunctions", "Derived": { "Tag":
"UNORDEREDLIST", "IsAnonymous": true, "UnorderedList": "string" }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "FeedPolicyRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PolicyName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Description", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Properties", "FieldType":
"FeedPolicyRecordType_Properties", "IsNullable": false } ] } }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"FeedPolicyRecordType_Properties", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList": "FeedPolicyRecordType_Properties_Item" },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"FeedPolicyRecordType_Properties_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"Name", "FieldType": "string", "IsNullable": false }, { "FieldName": "Value",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "FeedRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "FeedName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "AdapterName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "AdapterConfiguration", "FieldType":
"FeedRecordType_AdapterConfiguration", "IsNullable": false }, { "FieldName":
"Timestamp", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"FeedRecordType_AdapterConfiguration", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList":
"FeedRecordType_AdapterConfiguration_Item" }, "Timestamp": "Thu Mar 09 15:46:42
PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"FeedRecordType_AdapterConfiguration_Item", "Derived": { "Tag": "RECORD",
"IsAnonymous": true, "Record": { "IsOpen": true, "Fields": [ { "FieldName":
"Name", "FieldType": "string", "IsNullable": false }, { "FieldName": "Value",
"FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "FunctionRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "DataverseName", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Name", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Arity", "FieldType": "string",
"IsNullable": false }, { "FieldName": "Params", "FieldType":
"FunctionRecordType_Params", "IsNullable": false }, { "FieldName":
"ReturnType", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Definition", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Language", "FieldType": "string", "IsNullable": false }, { "FieldName":
"Kind", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "FunctionRecordType_Params",
"Derived": { "Tag": "ORDEREDLIST", "IsAnonymous": true, "OrderedList": "string"
}, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "IndexRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "DatasetName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "IndexName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "IndexStructure", "FieldType": "string", "IsNullable": false }, {
"FieldName": "SearchKey", "FieldType": "IndexRecordType_SearchKey",
"IsNullable": false }, { "FieldName": "IsPrimary", "FieldType": "boolean",
"IsNullable": false }, { "FieldName": "Timestamp", "FieldType": "string",
"IsNullable": false }, { "FieldName": "PendingOp", "FieldType": "int32",
"IsNullable": false } ] } }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "IndexRecordType_SearchKey",
"Derived": { "Tag": "ORDEREDLIST", "IsAnonymous": true, "OrderedList":
"IndexRecordType_SearchKey_Item" }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017"
}
+{ "DataverseName": "Metadata", "DatatypeName":
"IndexRecordType_SearchKey_Item", "Derived": { "Tag": "ORDEREDLIST",
"IsAnonymous": true, "OrderedList": "string" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "LibraryRecordType", "Derived":
{ "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields":
[ { "FieldName": "DataverseName", "FieldType": "string", "IsNullable": false },
{ "FieldName": "Name", "FieldType": "string", "IsNullable": false }, {
"FieldName": "Timestamp", "FieldType": "string", "IsNullable": false } ] } },
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "NodeGroupRecordType",
"Derived": { "Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true,
"Fields": [ { "FieldName": "GroupName", "FieldType": "string", "IsNullable":
false }, { "FieldName": "NodeNames", "FieldType":
"NodeGroupRecordType_NodeNames", "IsNullable": false }, { "FieldName":
"Timestamp", "FieldType": "string", "IsNullable": false } ] } }, "Timestamp":
"Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName":
"NodeGroupRecordType_NodeNames", "Derived": { "Tag": "UNORDEREDLIST",
"IsAnonymous": true, "UnorderedList": "string" }, "Timestamp": "Thu Mar 09
15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "NodeRecordType", "Derived": {
"Tag": "RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [
{ "FieldName": "NodeName", "FieldType": "string", "IsNullable": false }, {
"FieldName": "NumberOfCores", "FieldType": "int64", "IsNullable": false }, {
"FieldName": "WorkingMemorySize", "FieldType": "int64", "IsNullable": false } ]
} }, "Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "binary", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "boolean", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "circle", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "date", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "datetime", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "day-time-duration",
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "double", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "duration", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "float", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "int16", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "int32", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "int64", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "int8", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "interval", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "test", "DatatypeName": "kv1", "Derived": { "Tag":
"RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "id", "FieldType": "int32", "IsNullable": false }, { "FieldName":
"val", "FieldType": "AnyObject", "IsNullable": false } ] } }, "Timestamp": "Thu
Mar 09 15:46:46 PST 2017" }
+{ "DataverseName": "test", "DatatypeName": "kv2", "Derived": { "Tag":
"RECORD", "IsAnonymous": false, "Record": { "IsOpen": true, "Fields": [ {
"FieldName": "id", "FieldType": "int32", "IsNullable": false }, { "FieldName":
"val", "FieldType": "AnyObject", "IsNullable": false } ] } }, "Timestamp": "Thu
Mar 09 15:46:46 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "line", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "missing", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "null", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "point", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "point3d", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "polygon", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "rectangle", "Timestamp": "Thu
Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "shortwithouttypeinfo",
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "string", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "time", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "uuid", "Timestamp": "Thu Mar
09 15:46:42 PST 2017" }
+{ "DataverseName": "Metadata", "DatatypeName": "year-month-duration",
"Timestamp": "Thu Mar 09 15:46:42 PST 2017" }
diff --git
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/DatasetLifecycleManager.java
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/DatasetLifecycleManager.java
index 1a8ccae..05ad42e 100644
---
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/DatasetLifecycleManager.java
+++
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/DatasetLifecycleManager.java
@@ -133,7 +133,8 @@
PrimaryIndexOperationTracker opTracker = dsr.getOpTracker();
if (iInfo.getReferenceCount() != 0 || (opTracker != null &&
opTracker.getNumActiveOperations() != 0)) {
- throw new HyracksDataException("Cannot remove index while it is
open. (Dataset reference count = "
+ throw new HyracksDataException("Cannot remove index" +
iInfo.getIndex()
+ + " while it is open. (Dataset reference count = "
+ iInfo.getReferenceCount() + ", Operation tracker number
of active operations = "
+ opTracker.getNumActiveOperations() + ")");
}
@@ -196,6 +197,9 @@
iInfo.setOpen(true);
}
iInfo.touch();
+ if
(iInfo.getIndex().toString().endsWith("test/DBLP_idx_ngram_index/")) {
+ System.out.println("XXXX touch index " + iInfo.getIndex());
+ }
}
private boolean evictCandidateDataset() throws HyracksDataException {
@@ -266,19 +270,34 @@
@Override
public synchronized void close(String resourcePath) throws
HyracksDataException {
- validateDatasetLifecycleManagerState();
- int did = getDIDfromResourcePath(resourcePath);
- long resourceID = getResourceIDfromResourcePath(resourcePath);
- DatasetResource dsr = datasets.get(did);
- if (dsr == null) {
- throw new HyracksDataException("No index found with resourceID " +
resourceID);
+ DatasetResource dsr = null;
+ IndexInfo iInfo = null;
+ try {
+ validateDatasetLifecycleManagerState();
+ int did = getDIDfromResourcePath(resourcePath);
+ long resourceID = getResourceIDfromResourcePath(resourcePath);
+ dsr = datasets.get(did);
+ if (dsr == null) {
+ throw new HyracksDataException("No index found with resourceID
" + resourceID);
+ }
+ iInfo = dsr.getIndexInfo(resourceID);
+ if (iInfo == null) {
+ throw new HyracksDataException("No index found with resourceID
" + resourceID);
+ }
+ } finally {
+ try {
+ if (iInfo != null) {
+ iInfo.untouch();
+ if
(iInfo.getIndex().toString().endsWith("test/DBLP_idx_ngram_index/")) {
+ System.out.println("XXXX untouch index " +
iInfo.getIndex());
+ }
+ }
+ } finally {
+ if (dsr != null) {
+ dsr.untouch();
+ }
+ }
}
- IndexInfo iInfo = dsr.getIndexInfo(resourceID);
- if (iInfo == null) {
- throw new HyracksDataException("No index found with resourceID " +
resourceID);
- }
- iInfo.untouch();
- dsr.untouch();
}
@Override
diff --git
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/Info.java
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/Info.java
index 999eb34..8afae0d 100644
---
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/Info.java
+++
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/context/Info.java
@@ -39,10 +39,6 @@
return referenceCount;
}
- public void setReferenceCount(int referenceCount) {
- this.referenceCount = referenceCount;
- }
-
public boolean isOpen() {
return isOpen;
}
diff --git
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/utils/Servlets.java
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/utils/Servlets.java
index 5ffb334..df57ebd 100644
---
a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/utils/Servlets.java
+++
b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/utils/Servlets.java
@@ -34,6 +34,7 @@
public static final String CONNECTOR = "/connector";
public static final String SHUTDOWN = "/admin/shutdown";
public static final String VERSION = "/admin/version";
+ public static final String QUERY_CANCEL = "/admin/request/cancel/*";
public static final String CLUSTER_STATE = "/admin/cluster/*";
public static final String CLUSTER_STATE_NODE_DETAIL =
"/admin/cluster/node/*";
public static final String CLUSTER_STATE_CC_DETAIL = "/admin/cluster/cc/*";
diff --git
a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/operators/LSMPrimaryUpsertOperatorNodePushable.java
b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/operators/LSMPrimaryUpsertOperatorNodePushable.java
index 037945a..c75c252 100644
---
a/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/operators/LSMPrimaryUpsertOperatorNodePushable.java
+++
b/asterixdb/asterix-runtime/src/main/java/org/apache/asterix/runtime/operators/LSMPrimaryUpsertOperatorNodePushable.java
@@ -161,7 +161,6 @@
frameOpCallback =
frameOpCallbackFactory.createFrameOperationCallback(ctx,
(ILSMIndexAccessor) indexAccessor);
} catch (Exception e) {
- indexHelper.close();
throw new HyracksDataException(e);
}
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/FileHandle.java
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/FileHandle.java
index 33b8980..560c2a4 100644
---
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/FileHandle.java
+++
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/FileHandle.java
@@ -68,7 +68,6 @@
throw new IllegalArgumentException();
}
raf = new RandomAccessFile(fileRef.getFile(), mode);
- channel = raf.getChannel();
}
public void close() throws IOException {
@@ -80,10 +79,10 @@
}
public FileChannel getFileChannel() {
+ if (channel == null) {
+ channel = raf.getChannel();
+ }
return channel;
}
- public void sync(boolean metadata) throws IOException {
- channel.force(metadata);
- }
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/IOManager.java
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/IOManager.java
index 5ccdaa8..d97a7b5 100644
---
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/IOManager.java
+++
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/io/IOManager.java
@@ -324,7 +324,7 @@
@Override
public void sync(IFileHandle fileHandle, boolean metadata) throws
HyracksDataException {
try {
- ((FileHandle) fileHandle).sync(metadata);
+ ((FileHandle) fileHandle).getFileChannel().force(metadata);
} catch (IOException e) {
throw new HyracksDataException(e);
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/MaterializingPipelinedPartition.java
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/MaterializingPipelinedPartition.java
index 137ef37..cef3150 100644
---
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/MaterializingPipelinedPartition.java
+++
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/MaterializingPipelinedPartition.java
@@ -18,6 +18,7 @@
*/
package org.apache.hyracks.control.nc.partitions;
+import java.io.IOException;
import java.nio.ByteBuffer;
import java.util.concurrent.Executor;
import java.util.logging.Level;
@@ -51,7 +52,9 @@
private FileReference fRef;
- private IFileHandle handle;
+ private IFileHandle writeHandle;
+
+ private IFileHandle readHandle;
private long size;
@@ -60,6 +63,8 @@
private boolean failed;
protected boolean flushRequest;
+
+ private Thread pushDataThread;
private Level openCloseLevel = Level.FINE;
@@ -80,6 +85,13 @@
@Override
public void deallocate() {
+ if (readHandle != null) {
+ try {
+ ioManager.close(readHandle);
+ } catch (IOException e) {
+ LOGGER.log(Level.WARNING, e.getMessage(), e);
+ }
+ }
if (fRef != null) {
fRef.delete();
}
@@ -87,70 +99,69 @@
@Override
public void writeTo(final IFrameWriter writer) {
- executor.execute(new Runnable() {
+ Runnable dataPusher = new Runnable() {
@Override
public void run() {
try {
synchronized (MaterializingPipelinedPartition.this) {
- while (fRef == null && eos == false) {
+ while (readHandle == null && !eos && !failed) {
MaterializingPipelinedPartition.this.wait();
}
}
- IFileHandle fh = fRef == null ? null
- : ioManager.open(fRef,
IIOManager.FileReadWriteMode.READ_ONLY,
-
IIOManager.FileSyncMode.METADATA_ASYNC_DATA_ASYNC);
+ writer.open();
try {
- writer.open();
- try {
- if (fh != null) {
- long offset = 0;
- ByteBuffer buffer = ctx.allocateFrame();
- boolean fail = false;
- boolean done = false;
- while (!fail && !done) {
- synchronized
(MaterializingPipelinedPartition.this) {
- while (offset >= size && !eos &&
!failed) {
- if (flushRequest) {
- flushRequest = false;
- writer.flush();
- }
- try {
-
MaterializingPipelinedPartition.this.wait();
- } catch (InterruptedException e) {
- throw new
HyracksDataException(e);
- }
- }
- flushRequest = false;
- fail = failed;
- done = eos && offset >= size;
+ long offset = 0;
+ ByteBuffer buffer = ctx.allocateFrame();
+ boolean done = false;
+ while (!done) {
+ boolean flush;
+ boolean fail;
+ synchronized
(MaterializingPipelinedPartition.this) {
+ while (offset >= size && !eos && !failed) {
+
MaterializingPipelinedPartition.this.wait();
}
- if (fail) {
- writer.fail();
- } else if (!done) {
- buffer.clear();
- long readLen = ioManager.syncRead(fh,
offset, buffer);
- if (readLen < buffer.capacity()) {
- throw new
HyracksDataException("Premature end of file");
- }
- offset += readLen;
- buffer.flip();
- writer.nextFrame(buffer);
+ flush = flushRequest;
+ flushRequest = false; // Clear the flush flag.
+ fail = failed;
+ done = eos && offset >= size;
+ }
+ if (fail) {
+ writer.fail();
+ break;
+ }
+ if (!done) {
+ buffer.clear();
+ long readLen = ioManager.syncRead(readHandle,
offset, buffer);
+ if (readLen < buffer.capacity()) {
+ throw new HyracksDataException("Premature
end of file");
}
+ offset += readLen;
+ buffer.flip();
+ writer.nextFrame(buffer);
}
+ if (flush) {
+ writer.flush();
+ }
+ }
+ } catch (Exception e) {
+ writer.fail();
+ throw e;
+ } finally {
+ try {
+ if (readHandle != null) {
+ ioManager.close(readHandle); // close is
idempotent.
}
} finally {
writer.close();
}
- } finally {
- if (fh != null) {
- ioManager.close(fh);
- }
}
} catch (Exception e) {
- throw new RuntimeException(e);
+ LOGGER.log(Level.SEVERE, e.getMessage(), e);
}
}
- });
+ };
+ pushDataThread = new Thread(dataPusher);
+ executor.execute(pushDataThread);
}
@Override
@@ -172,7 +183,9 @@
private void checkOrCreateFile() throws HyracksDataException {
if (fRef == null) {
fRef =
manager.getFileFactory().createUnmanagedWorkspaceFile(pid.toString().replace(":",
"$"));
- handle = ctx.getIOManager().open(fRef,
IIOManager.FileReadWriteMode.READ_WRITE,
+ writeHandle = ioManager.open(fRef,
IIOManager.FileReadWriteMode.READ_WRITE,
+ IIOManager.FileSyncMode.METADATA_ASYNC_DATA_ASYNC);
+ readHandle = ioManager.open(fRef,
IIOManager.FileReadWriteMode.READ_ONLY,
IIOManager.FileSyncMode.METADATA_ASYNC_DATA_ASYNC);
}
}
@@ -180,7 +193,7 @@
@Override
public synchronized void nextFrame(ByteBuffer buffer) throws
HyracksDataException {
checkOrCreateFile();
- size += ctx.getIOManager().syncWrite(handle, size, buffer);
+ size += ctx.getIOManager().syncWrite(writeHandle, size, buffer);
notifyAll();
}
@@ -195,12 +208,12 @@
if (LOGGER.isLoggable(openCloseLevel)) {
LOGGER.log(openCloseLevel, "close(" + pid + " by " + taId);
}
+ if (writeHandle != null) {
+ ctx.getIOManager().close(writeHandle);
+ }
synchronized (this) {
eos = true;
- if (handle != null) {
- ctx.getIOManager().close(handle);
- }
- handle = null;
+ writeHandle = null;
notifyAll();
}
}
@@ -210,4 +223,5 @@
flushRequest = true;
notifyAll();
}
+
}
\ No newline at end of file
diff --git
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/PipelinedPartition.java
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/PipelinedPartition.java
index 84d2283..fd434d7 100644
---
a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/PipelinedPartition.java
+++
b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/partitions/PipelinedPartition.java
@@ -76,12 +76,12 @@
manager.registerPartition(pid, taId, this, PartitionState.STARTED,
false);
failed = false;
pendingConnection = true;
+ ensureConnected();
}
@Override
public void nextFrame(ByteBuffer buffer) throws HyracksDataException {
if (!failed) {
- ensureConnected();
delegate.nextFrame(buffer);
}
}
@@ -113,7 +113,6 @@
@Override
public void close() throws HyracksDataException {
if (!failed) {
- ensureConnected();
delegate.close();
}
}
@@ -121,7 +120,6 @@
@Override
public void flush() throws HyracksDataException {
if (!failed) {
- ensureConnected();
delegate.flush();
}
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/RunFileReader.java
b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/RunFileReader.java
index f0bd318..ffee1a6 100644
---
a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/RunFileReader.java
+++
b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/RunFileReader.java
@@ -84,6 +84,9 @@
@Override
public void close() throws HyracksDataException {
+ if (handle == null) {
+ return; // Makes sure the close operation is idempotent.
+ }
if (deleteAfterClose) {
try {
ioManager.close(handle);
@@ -94,6 +97,7 @@
} else {
ioManager.close(handle);
}
+ handle = null;
}
public long getFileSize() {
diff --git
a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/connectors/PartitionDataWriter.java
b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/connectors/PartitionDataWriter.java
index dbd3afa..0ac8cec 100644
---
a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/connectors/PartitionDataWriter.java
+++
b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/connectors/PartitionDataWriter.java
@@ -74,19 +74,29 @@
@Override
public void close() throws HyracksDataException {
HyracksDataException closeException = null;
+ if (!failed) {
+ boolean newFailure = false;
+ for (int i = 0; i < pWriters.length; ++i) {
+ try {
+ if (isOpen[i] && allocatedFrames[i] &&
appenders[i].getTupleCount() > 0) {
+ appenders[i].write(pWriters[i], true);
+ }
+ } catch (Exception e) {
+ newFailure = true;
+ closeException = HyracksDataException.create(e);
+ break;
+ }
+ }
+ if (newFailure) {
+ fail(); // Fail all writers if any new failure happens.
+ }
+ }
for (int i = 0; i < pWriters.length; ++i) {
if (isOpen[i]) {
- if (allocatedFrames[i] && appenders[i].getTupleCount() > 0 &&
!failed) {
- try {
- appenders[i].write(pWriters[i], true);
- } catch (Throwable th) {
- closeException =
HyracksDataException.suppress(closeException, th);
- }
- }
try {
pWriters[i].close();
- } catch (Throwable th) {
- closeException =
HyracksDataException.suppress(closeException, th);
+ } catch (Exception e) {
+ closeException = HyracksDataException.create(e);
}
}
}
@@ -129,14 +139,15 @@
if (isOpen[i]) {
try {
pWriters[i].fail();
- } catch (Throwable th) {
- failException =
HyracksDataException.suppress(failException, th);
+ } catch (Exception e) {
+ failException = HyracksDataException.create(e);
}
}
}
if (failException != null) {
throw failException;
}
+ failed = true;
}
@Override
diff --git
a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
index f4158ac..89238e5 100644
---
a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
+++
b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
@@ -35,6 +35,7 @@
import org.apache.hyracks.api.exceptions.HyracksDataException;
import org.apache.hyracks.dataflow.common.comm.util.FrameUtils;
import org.apache.hyracks.dataflow.common.io.GeneratedRunFileReader;
+import org.apache.hyracks.dataflow.common.io.RunFileReader;
import org.apache.hyracks.dataflow.common.io.RunFileWriter;
import org.apache.hyracks.dataflow.std.sort.util.GroupVSizeFrame;
@@ -167,11 +168,18 @@
if (finalWriter != null) {
finalWriter.fail();
}
- throw new HyracksDataException(e);
+ throw HyracksDataException.create(e);
} finally {
- if (finalWriter != null) {
- finalWriter.close();
+ try {
+ for (RunFileReader reader : runs) {
+ reader.close(); // close is idempotent.
+ }
+ } finally {
+ if (finalWriter != null) {
+ finalWriter.close();
+ }
}
+
}
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexDataflowHelper.java
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexDataflowHelper.java
index c6d4e35..40366da 100644
---
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexDataflowHelper.java
+++
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexDataflowHelper.java
@@ -123,6 +123,7 @@
}
}
+
@Override
public void close() throws HyracksDataException {
synchronized (lcManager) {
diff --git
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexSearchOperatorNodePushable.java
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexSearchOperatorNodePushable.java
index c089854..09823aa 100644
---
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexSearchOperatorNodePushable.java
+++
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/IndexSearchOperatorNodePushable.java
@@ -71,6 +71,7 @@
protected final int[] maxFilterFieldIndexes;
protected PermutingFrameTupleReference minFilterKey;
protected PermutingFrameTupleReference maxFilterKey;
+ private boolean failed = false;
public IndexSearchOperatorNodePushable(IIndexOperatorDescriptor opDesc,
IHyracksTaskContext ctx, int partition,
IRecordDescriptorProvider recordDescProvider, int[]
minFilterFieldIndexes, int[] maxFilterFieldIndexes)
@@ -196,53 +197,50 @@
@Override
public void close() throws HyracksDataException {
- HyracksDataException closeException = null;
- if (index != null) {
- // if index == null, then the index open was not successful
+ try {
+ HyracksDataException closeException = null;
+ if (index == null) {
+ // if index == null, then the index open was not successful
+ return;
+ }
try {
- if (appender.getTupleCount() > 0) {
+ if (appender.getTupleCount() > 0 && !failed) {
appender.write(writer, true);
}
- } catch (Throwable th) {
- closeException = new HyracksDataException(th);
+ } catch (Exception e) {
+ closeException = HyracksDataException.create(e);
}
try {
cursor.close();
- } catch (Throwable th) {
+ } catch (Exception e) {
if (closeException == null) {
- closeException = new HyracksDataException(th);
+ closeException = HyracksDataException.create(e);
} else {
- closeException.addSuppressed(th);
+ closeException.addSuppressed(e);
}
}
+
try {
indexHelper.close();
- } catch (Throwable th) {
+ } catch (Exception e) {
if (closeException == null) {
- closeException = new HyracksDataException(th);
+ closeException = HyracksDataException.create(e);
} else {
- closeException.addSuppressed(th);
+ closeException.addSuppressed(e);
}
}
- }
- try {
- // will definitely be called regardless of exceptions
- writer.close();
- } catch (Throwable th) {
- if (closeException == null) {
- closeException = new HyracksDataException(th);
- } else {
- closeException.addSuppressed(th);
+ if (closeException != null) {
+ throw closeException;
}
- }
- if (closeException != null) {
- throw closeException;
+ } finally {
+ writer.close();
}
}
@Override
public void fail() throws HyracksDataException {
writer.fail();
+ failed = true;
}
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/TreeIndexStatsOperatorNodePushable.java
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/TreeIndexStatsOperatorNodePushable.java
index 91d433c..4e6c6ad 100644
---
a/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/TreeIndexStatsOperatorNodePushable.java
+++
b/hyracks-fullstack/hyracks/hyracks-storage-am-common/src/main/java/org/apache/hyracks/storage/am/common/dataflow/TreeIndexStatsOperatorNodePushable.java
@@ -94,8 +94,11 @@
writer.fail();
throw new HyracksDataException(e);
} finally {
- writer.close();
- treeIndexHelper.close();
+ try {
+ treeIndexHelper.close();
+ } finally {
+ writer.close();
+ }
}
}
}
diff --git
a/hyracks-fullstack/hyracks/hyracks-storage-am-lsm-common/src/main/java/org/apache/hyracks/storage/am/lsm/common/dataflow/LSMIndexCompactOperatorNodePushable.java
b/hyracks-fullstack/hyracks/hyracks-storage-am-lsm-common/src/main/java/org/apache/hyracks/storage/am/lsm/common/dataflow/LSMIndexCompactOperatorNodePushable.java
index 48f59f1..b787994 100644
---
a/hyracks-fullstack/hyracks/hyracks-storage-am-lsm-common/src/main/java/org/apache/hyracks/storage/am/lsm/common/dataflow/LSMIndexCompactOperatorNodePushable.java
+++
b/hyracks-fullstack/hyracks/hyracks-storage-am-lsm-common/src/main/java/org/apache/hyracks/storage/am/lsm/common/dataflow/LSMIndexCompactOperatorNodePushable.java
@@ -62,8 +62,9 @@
try {
accessor.scheduleFullMerge(NoOpIOOperationCallback.INSTANCE);
} catch (Exception e) {
+ throw HyracksDataException.create(e);
+ } finally {
indexHelper.close();
- throw new HyracksDataException(e);
}
}
--
To view, visit https://asterix-gerrit.ics.uci.edu/1564
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings
Gerrit-MessageType: newchange
Gerrit-Change-Id: I2936ac83f71bbef533e2695ed0a2b220c23fc483
Gerrit-PatchSet: 1
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Yingyi Bu <[email protected]>