This is an automated email from the ASF dual-hosted git repository.
zstan pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/ignite-3.git
The following commit(s) were added to refs/heads/main by this push:
new cee8310315 IGNITE-17953 Sql. NPE on some malformed queries - Fixes
#1345.
cee8310315 is described below
commit cee831031501c36d1d24b1dd72a84747be8f18f6
Author: zstan <[email protected]>
AuthorDate: Fri Nov 25 09:49:34 2022 +0300
IGNITE-17953 Sql. NPE on some malformed queries - Fixes #1345.
Signed-off-by: zstan <[email protected]>
---
.../client/io/netty/NettyClientConnection.java | 4 +--
.../runner/app/jdbc/ItJdbcErrorsSelfTest.java | 40 ++++++++++++++++++++++
.../internal/sql/engine/SqlQueryProcessor.java | 29 ++++++++++++++--
3 files changed, 68 insertions(+), 5 deletions(-)
diff --git
a/modules/client/src/main/java/org/apache/ignite/internal/client/io/netty/NettyClientConnection.java
b/modules/client/src/main/java/org/apache/ignite/internal/client/io/netty/NettyClientConnection.java
index d21f175b52..a834318d99 100644
---
a/modules/client/src/main/java/org/apache/ignite/internal/client/io/netty/NettyClientConnection.java
+++
b/modules/client/src/main/java/org/apache/ignite/internal/client/io/netty/NettyClientConnection.java
@@ -21,7 +21,6 @@ import io.netty.buffer.ByteBuf;
import io.netty.channel.Channel;
import io.netty.channel.ChannelFuture;
import io.netty.util.AttributeKey;
-import java.io.IOException;
import java.net.InetSocketAddress;
import org.apache.ignite.internal.client.io.ClientConnection;
import org.apache.ignite.internal.client.io.ClientConnectionStateHandler;
@@ -88,9 +87,8 @@ public class NettyClientConnection implements
ClientConnection {
* Handles incoming message.
*
* @param buf Message.
- * @throws IOException when message can't be decoded.
*/
- void onMessage(ByteBuf buf) throws IOException {
+ void onMessage(ByteBuf buf) {
msgHnd.onMessage(buf);
}
diff --git
a/modules/runner/src/integrationTest/java/org/apache/ignite/internal/runner/app/jdbc/ItJdbcErrorsSelfTest.java
b/modules/runner/src/integrationTest/java/org/apache/ignite/internal/runner/app/jdbc/ItJdbcErrorsSelfTest.java
index d15d1eed71..f10d2284c1 100644
---
a/modules/runner/src/integrationTest/java/org/apache/ignite/internal/runner/app/jdbc/ItJdbcErrorsSelfTest.java
+++
b/modules/runner/src/integrationTest/java/org/apache/ignite/internal/runner/app/jdbc/ItJdbcErrorsSelfTest.java
@@ -22,6 +22,7 @@ import static
org.apache.ignite.internal.jdbc.proto.SqlStateCode.INVALID_TRANSAC
import static
org.apache.ignite.internal.jdbc.proto.SqlStateCode.UNSUPPORTED_OPERATION;
import static org.junit.jupiter.api.Assertions.assertArrayEquals;
import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertThrows;
import static org.junit.jupiter.api.Assertions.assertTrue;
import static org.junit.jupiter.api.Assertions.fail;
@@ -29,6 +30,7 @@ import java.sql.BatchUpdateException;
import java.sql.Connection;
import java.sql.DriverManager;
import java.sql.SQLException;
+import java.sql.Statement;
import org.junit.jupiter.api.Disabled;
import org.junit.jupiter.api.Test;
@@ -51,6 +53,44 @@ public class ItJdbcErrorsSelfTest extends
ItJdbcErrorsAbstractSelfTest {
CLIENT_CONNECTION_FAILED, "Failed to connect to server");
}
+ /**
+ * Test that execution of erroneous queries are not stopping execution.
+ * Also check correctness of exception messages.
+ *
+ * @throws SQLException If connection can`t be established.
+ */
+ @Test
+ public void processMixedQueries() throws SQLException {
+ conn =
DriverManager.getConnection("jdbc:ignite:thin://127.0.0.1:10800/");
+
+ try (Statement stmt = conn.createStatement()) {
+ stmt.execute(
+ "CREATE TABLE CITIES ("
+ + "ID INT PRIMARY KEY,"
+ + "NAME VARCHAR)"
+ );
+
+ SQLException ex = assertThrows(SQLException.class, () ->
stmt.execute("non sql stuff"));
+
+ assertTrue(ex.getMessage().contains("Failed to parse query"));
+ }
+
+ try (Statement stmt = conn.createStatement()) {
+ stmt.execute(
+ "CREATE TABLE ACCOUNTS ("
+ + " ACCOUNT_ID INT PRIMARY KEY,"
+ + " CITY_ID INT,"
+ + " FIRST_NAME VARCHAR,"
+ + " LAST_NAME VARCHAR,"
+ + " BALANCE DOUBLE)"
+ );
+
+ SQLException ex = assertThrows(SQLException.class, () ->
stmt.execute("CREATE TABLE ACCOUNTS (ACCOUNT_ID INT PRIMARY KEY)"));
+
+ assertTrue(ex.getMessage().contains("Table already exists"));
+ }
+ }
+
/**
* Test error code for the case when connection string is a mess.
*/
diff --git
a/modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java
b/modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java
index 8491b3ee99..fd74d12688 100644
---
a/modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java
+++
b/modules/sql-engine/src/main/java/org/apache/ignite/internal/sql/engine/SqlQueryProcessor.java
@@ -473,11 +473,19 @@ public class SqlQueryProcessor implements QueryProcessor {
throw new IgniteInternalException(SCHEMA_NOT_FOUND_ERR,
format("Schema not found [schemaName={}]", schemaName));
}
- SqlNodeList nodes = Commons.parse(sql,
FRAMEWORK_CONFIG.getParserConfig());
+ CompletableFuture<Void> start = new CompletableFuture<>();
+
+ SqlNodeList nodes = SqlNodeList.EMPTY;
var res = new
ArrayList<CompletableFuture<AsyncSqlCursor<List<Object>>>>(nodes.size());
- CompletableFuture<Void> start = new CompletableFuture<>();
+ try {
+ nodes = Commons.parse(sql, FRAMEWORK_CONFIG.getParserConfig());
+ } catch (Throwable th) {
+ start.completeExceptionally(th);
+
+ res.add(CompletableFuture.completedFuture(failedCursor(th)));
+ }
for (SqlNode sqlNode : nodes) {
boolean needStartTx = SqlKind.DML.contains(sqlNode.getKind()) ||
SqlKind.QUERY.contains(sqlNode.getKind());
@@ -526,6 +534,23 @@ public class SqlQueryProcessor implements QueryProcessor {
return res;
}
+ private static <T> AsyncSqlCursor<T> failedCursor(Throwable th) {
+ return new AsyncSqlCursorImpl<>(
+ null, null, null,
+ new AsyncCursor<>() {
+ @Override
+ public CompletableFuture<BatchedResult<T>>
requestNextAsync(int rows) {
+ return CompletableFuture.failedFuture(th);
+ }
+
+ @Override
+ public CompletableFuture<Void> closeAsync() {
+ return CompletableFuture.completedFuture(null);
+ }
+ }
+ );
+ }
+
private abstract static class AbstractTableEventListener implements
EventListener<TableEventParameters> {
protected final SqlSchemaManagerImpl schemaHolder;