xtern commented on code in PR #2965:
URL: https://github.com/apache/ignite-3/pull/2965#discussion_r1432833012


##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -835,7 +851,42 @@ CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
processNext() {
                         // Try to rollback script managed transaction, if any.
                         txCtx.rollbackUncommitted();
                     } else {
-                        taskExecutor.execute(this::processNext);
+                        CompletableFuture<Void> triggerFuture;
+                        ScriptStatement nextStatement = statements.peek();
+
+                        if (txWrapper == null) {
+                            // tx is started already, no need to wait
+                            triggerFuture = nullCompletedFuture();
+                        } else if (txWrapper.implicit()) {
+                            if (cursor.queryType() != SqlQueryType.QUERY) {
+                                // any query apart from type of QUERY returns 
at most a single row, so
+                                // it should be safe to commit transaction 
prematurely after receiving
+                                // `firstPageReady` signal, since all sources 
have been processed
+                                triggerFuture = 
cursor.onFirstPageReady().thenCompose(none -> txWrapper.commitImplicit());
+                            } else if (nextStatement != null
+                                    && 
readOnlyQuery(nextStatement.parsedResult.queryType())) {

Review Comment:
   why do we need this condition (why do we need to wait for prefetch of SELECT 
if the next statement is read-write)?
   Do we have a test that shows that it is necessary?



##########
modules/sql-engine/src/integrationTest/java/org/apache/ignite/internal/sql/engine/ItSqlMultiStatementTxTest.java:
##########
@@ -230,22 +230,18 @@ void dmlScriptRollsBackImplicitly() throws 
InterruptedException {
 
     @Test
     void openedScriptTransactionRollsBackOnError() {
-        {
-            AsyncSqlCursor<InternalSqlRow> cursor = runScript(
-                    "START TRANSACTION READ WRITE;"
-                    + "INSERT INTO test VALUES(2);"
-                    + "INSERT INTO test VALUES(2/0);"
-                    + "SELECT 1;"
-                    + "COMMIT;"
-            );
+        String script = "START TRANSACTION READ WRITE;"
+                + "INSERT INTO test VALUES(2);"
+                + "INSERT INTO test VALUES(2/0);"

Review Comment:
   what is the expected behavior if SELECT will raise an error?
   Should script be executed completely?
   
   From my experiments (may be wrong) I see that sometime the script completes 
with error, but sometimes without, my be we need to not interrupt script 
execution in case of errors from  SELECT statements at all.



##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -619,17 +622,7 @@ private CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
executeParsedStatement
                             .build();
 
                     CompletableFuture<AsyncSqlCursor<InternalSqlRow>> fut = 
prepareSvc.prepareAsync(parsedResult, ctx)

Review Comment:
   the `fut` variable now looks redundant



##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -835,7 +851,42 @@ CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
processNext() {
                         // Try to rollback script managed transaction, if any.
                         txCtx.rollbackUncommitted();
                     } else {
-                        taskExecutor.execute(this::processNext);
+                        CompletableFuture<Void> triggerFuture;
+                        ScriptStatement nextStatement = statements.peek();
+
+                        if (txWrapper == null) {
+                            // tx is started already, no need to wait
+                            triggerFuture = nullCompletedFuture();
+                        } else if (txWrapper.implicit()) {
+                            if (cursor.queryType() != SqlQueryType.QUERY) {
+                                // any query apart from type of QUERY returns 
at most a single row, so
+                                // it should be safe to commit transaction 
prematurely after receiving
+                                // `firstPageReady` signal, since all sources 
have been processed
+                                triggerFuture = 
cursor.onFirstPageReady().thenCompose(none -> txWrapper.commitImplicit());

Review Comment:
   As I understand, for non-QUERY we can't get the cursor until the 
`onFirstPageReady()` completes.
   Can we simplify this to
   ```
   triggerFuture = txWrapper.commitImplicit();
   ```
   ?



##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -656,9 +649,10 @@ private CompletableFuture<SchemaPlus> 
waitForActualSchema(String schemaName, Hyb
         }
     }
 
-    private AsyncSqlCursor<InternalSqlRow> executePlan(
+    private CompletableFuture<AsyncSqlCursor<InternalSqlRow>> executePlan(
             QueryTransactionWrapper txWrapper,
             BaseQueryContext ctx,
+            PrefetchCallback callback,

Review Comment:
   We can get it from BaseQueryContext



##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -835,7 +851,42 @@ CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
processNext() {
                         // Try to rollback script managed transaction, if any.
                         txCtx.rollbackUncommitted();
                     } else {
-                        taskExecutor.execute(this::processNext);
+                        CompletableFuture<Void> triggerFuture;
+                        ScriptStatement nextStatement = statements.peek();
+
+                        if (txWrapper == null) {
+                            // tx is started already, no need to wait
+                            triggerFuture = nullCompletedFuture();
+                        } else if (txWrapper.implicit()) {
+                            if (cursor.queryType() != SqlQueryType.QUERY) {
+                                // any query apart from type of QUERY returns 
at most a single row, so
+                                // it should be safe to commit transaction 
prematurely after receiving
+                                // `firstPageReady` signal, since all sources 
have been processed
+                                triggerFuture = 
cursor.onFirstPageReady().thenCompose(none -> txWrapper.commitImplicit());
+                            } else if (nextStatement != null
+                                    && 
readOnlyQuery(nextStatement.parsedResult.queryType())) {
+                                // if next statement only reads data, no need 
to wait
+                                triggerFuture = nullCompletedFuture();
+                            } else {
+                                triggerFuture = cursor.onFirstPageReady();
+                            }
+                        } else {
+                            if (readOnlyQuery(cursor.queryType())
+                                    && nextStatement != null
+                                    && 
readOnlyQuery(nextStatement.parsedResult.queryType())) {

Review Comment:
   The same question - why we need to wait for the prefetch of the last SELECT?
   Even if this is necessary - we can't be sure that the prefetch for the 
previous SELECT has already been done, so it looks strange.
   
   for example 
   ```
   SELECT 1
   SELECT 2
   INSERT ...
   ```
   
   as I understand we are waiting here for prefetch of SELECT 2, but not for 
SELECT 1, in theory prefetch for SELECT 2 can finishes before prefetch for 
SELECT 1



##########
modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java:
##########
@@ -603,12 +607,11 @@ private CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
executeParsedStatement
             QueryTransactionWrapper txWrapper,
             QueryCancel queryCancel,
             Object[] params,
-            boolean waitForPrefetch,
             @Nullable CompletableFuture<AsyncSqlCursor<InternalSqlRow>> 
nextStatement
     ) {
         return waitForActualSchema(schemaName, 
txWrapper.unwrap().startTimestamp())
                 .thenCompose(schema -> {
-                    PrefetchCallback callback = waitForPrefetch ? new 
PrefetchCallback() : null;
+                    PrefetchCallback callback = new PrefetchCallback();

Review Comment:
   May be it worth to revise usage of this class in a separate ticket.
   At least we can "more" implicitly create it and/or replace it with a 
completable future, cleanup null checks and may be remove it from 
`basequerycontext` :thinking: 



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

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to