This is an automated email from the ASF dual-hosted git repository.
jlewandowski pushed a commit to branch cep-15-accord
in repository https://gitbox.apache.org/repos/asf/cassandra.git
The following commit(s) were added to refs/heads/cep-15-accord by this push:
new 6989e93074 Fix statement validation against partition range queries
6989e93074 is described below
commit 6989e93074cd2cde6d51822ec6577e8c7d3c6f71
Author: Jacek Lewandowski <[email protected]>
AuthorDate: Thu Feb 16 20:46:21 2023 +0100
Fix statement validation against partition range queries
patch by Jacek Lewandowski; reviewed by Caleb Rackliffe for CASSANDRA-18240
---
CHANGES.txt | 1 +
.../cql3/restrictions/StatementRestrictions.java | 20 +++++++++----
.../cassandra/cql3/statements/DeleteStatement.java | 2 +-
.../cql3/statements/ModificationStatement.java | 2 +-
.../cql3/statements/TransactionStatement.java | 33 +++++++++++++---------
.../cql3/statements/TransactionStatementTest.java | 28 +++++++++++++++++-
6 files changed, 65 insertions(+), 21 deletions(-)
diff --git a/CHANGES.txt b/CHANGES.txt
index 8058edab9f..0586c9f745 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
accord
+ * Fix statement validation against partition range queries (CASSANDRA-18240)
* Fix null value handling for static columns (CASSANDRA-18241)
* Feature Flag for Accord Transactions (CASSANDRA-18195)
* CEP-15: Multi-Partition Transaction CQL Support (Alpha) (CASSANDRA-17719)
diff --git
a/src/java/org/apache/cassandra/cql3/restrictions/StatementRestrictions.java
b/src/java/org/apache/cassandra/cql3/restrictions/StatementRestrictions.java
index 371bc589c3..1f590882f8 100644
--- a/src/java/org/apache/cassandra/cql3/restrictions/StatementRestrictions.java
+++ b/src/java/org/apache/cassandra/cql3/restrictions/StatementRestrictions.java
@@ -832,13 +832,23 @@ public final class StatementRestrictions
*
* @return <code>true</code> if all the primary key columns are restricted
by an equality relation.
*/
- public boolean hasAllPKColumnsRestrictedByEqualities()
+ public boolean hasAllPrimaryKeyColumnsRestrictedByEqualities()
+ {
+ return hasAllPartitionKeyColumnsRestrictedByEqualities()
+ && !hasUnrestrictedClusteringColumns()
+ &&
(clusteringColumnsRestrictions.hasOnlyEqualityRestrictions());
+ }
+
+ /**
+ * Checks that all the partition key columns are restricted by an equality
relation ('=' or 'IN').
+ *
+ * @return <code>true</code> if all the partition key columns are
restricted by an equality relation.
+ */
+ public boolean hasAllPartitionKeyColumnsRestrictedByEqualities()
{
return !isPartitionKeyRestrictionsOnToken()
- &&
!partitionKeyRestrictions.hasUnrestrictedPartitionKeyComponents(table)
- && (partitionKeyRestrictions.hasOnlyEqualityRestrictions())
- && !hasUnrestrictedClusteringColumns()
- &&
(clusteringColumnsRestrictions.hasOnlyEqualityRestrictions());
+ &&
!partitionKeyRestrictions.hasUnrestrictedPartitionKeyComponents(table)
+ && (partitionKeyRestrictions.hasOnlyEqualityRestrictions());
}
/**
diff --git a/src/java/org/apache/cassandra/cql3/statements/DeleteStatement.java
b/src/java/org/apache/cassandra/cql3/statements/DeleteStatement.java
index fff6dd33df..a8d99e588b 100644
--- a/src/java/org/apache/cassandra/cql3/statements/DeleteStatement.java
+++ b/src/java/org/apache/cassandra/cql3/statements/DeleteStatement.java
@@ -176,7 +176,7 @@ public class DeleteStatement extends ModificationStatement
conditions,
attrs);
- if (stmt.hasConditions() &&
!restrictions.hasAllPKColumnsRestrictedByEqualities())
+ if (stmt.hasConditions() &&
!restrictions.hasAllPrimaryKeyColumnsRestrictedByEqualities())
{
checkFalse(stmt.isVirtual(), "DELETE statements must restrict
all PRIMARY KEY columns with equality relations");
diff --git
a/src/java/org/apache/cassandra/cql3/statements/ModificationStatement.java
b/src/java/org/apache/cassandra/cql3/statements/ModificationStatement.java
index 41ae242cf9..a62921bf11 100644
--- a/src/java/org/apache/cassandra/cql3/statements/ModificationStatement.java
+++ b/src/java/org/apache/cassandra/cql3/statements/ModificationStatement.java
@@ -1149,7 +1149,7 @@ public abstract class ModificationStatement implements
CQLStatement.SingleKeyspa
public SelectStatement createSelectForTxn()
{
// TODO: get working with static-only updates that don't specify
any/all primary key columns
-
Preconditions.checkState(getRestrictions().hasAllPKColumnsRestrictedByEqualities());
+
Preconditions.checkState(getRestrictions().hasAllPrimaryKeyColumnsRestrictedByEqualities());
Selection selection = Selection.forColumns(metadata,
Lists.newArrayList(requiresRead), false);
return new SelectStatement(metadata,
bindVariables,
diff --git
a/src/java/org/apache/cassandra/cql3/statements/TransactionStatement.java
b/src/java/org/apache/cassandra/cql3/statements/TransactionStatement.java
index 0348ab9618..71f67c2f66 100644
--- a/src/java/org/apache/cassandra/cql3/statements/TransactionStatement.java
+++ b/src/java/org/apache/cassandra/cql3/statements/TransactionStatement.java
@@ -86,13 +86,14 @@ public class TransactionStatement implements CQLStatement
private static final Logger logger =
LoggerFactory.getLogger(TransactionStatement.class);
public static final String DUPLICATE_TUPLE_NAME_MESSAGE = "The name '%s'
has already been used by a LET assignment.";
- public static final String INCOMPLETE_PRIMARY_KEY_LET_MESSAGE = "SELECT in
LET assignment without LIMIT 1 must specify all primary key elements; CQL %s";
- public static final String INCOMPLETE_PRIMARY_KEY_SELECT_MESSAGE = "Normal
SELECT without LIMIT 1 must specify all primary key elements; CQL %s";
+ public static final String INCOMPLETE_PRIMARY_KEY_LET_MESSAGE = "SELECT in
LET assignment must specify either all primary key elements or all partition
key elements and LIMIT 1. In both cases partition key elements must be always
specified with equality operators; CQL %s";
+ public static final String INCOMPLETE_PRIMARY_KEY_SELECT_MESSAGE = "Normal
SELECT must specify either all primary key elements or all partition key
elements and LIMIT 1. In both cases partition key elements must be always
specified with equality operators; CQL %s";
public static final String NO_CONDITIONS_IN_UPDATES_MESSAGE = "Updates
within transactions may not specify their own conditions.";
public static final String NO_TIMESTAMPS_IN_UPDATES_MESSAGE = "Updates
within transactions may not specify custom timestamps.";
public static final String EMPTY_TRANSACTION_MESSAGE = "Transaction
contains no reads or writes";
public static final String SELECT_REFS_NEED_COLUMN_MESSAGE = "SELECT
references must specify a column.";
public static final String TRANSACTIONS_DISABLED_MESSAGE = "Accord
transactions are disabled. (See accord_transactions_enabled in cassandra.yaml)";
+ public static final String ILLEGAL_RANGE_QUERY_MESSAGE = "Range queries
are not allowed for reads within a transaction";
static class NamedSelect
{
@@ -167,10 +168,11 @@ public class TransactionStatement implements CQLStatement
statement.validate(state);
}
- TxnNamedRead createNamedRead(NamedSelect namedSelect, QueryOptions options)
+ TxnNamedRead createNamedRead(NamedSelect namedSelect, QueryOptions
options, ClientState state)
{
SelectStatement select = namedSelect.select;
ReadQuery readQuery = select.getQuery(options, 0);
+ checkTrue(readQuery instanceof SinglePartitionReadQuery.Group,
ILLEGAL_RANGE_QUERY_MESSAGE, select.asCQL(options, state));
// We reject reads from both LET and SELECT that do not specify a
single row.
@SuppressWarnings("unchecked")
@@ -182,10 +184,11 @@ public class TransactionStatement implements CQLStatement
return new TxnNamedRead(namedSelect.name,
Iterables.getOnlyElement(selectQuery.queries));
}
- List<TxnNamedRead> createNamedReads(NamedSelect namedSelect, QueryOptions
options)
+ List<TxnNamedRead> createNamedReads(NamedSelect namedSelect, QueryOptions
options, ClientState state)
{
SelectStatement select = namedSelect.select;
ReadQuery readQuery = select.getQuery(options, 0);
+ checkTrue(readQuery instanceof SinglePartitionReadQuery.Group,
ILLEGAL_RANGE_QUERY_MESSAGE, select.asCQL(options, state));
// We reject reads from both LET and SELECT that do not specify a
single row.
@SuppressWarnings("unchecked")
@@ -200,20 +203,20 @@ public class TransactionStatement implements CQLStatement
return list;
}
- private List<TxnNamedRead> createNamedReads(QueryOptions options,
Consumer<Key> keyConsumer)
+ private List<TxnNamedRead> createNamedReads(QueryOptions options,
ClientState state, Consumer<Key> keyConsumer)
{
List<TxnNamedRead> reads = new ArrayList<>(assignments.size() + 1);
for (NamedSelect select : assignments)
{
- TxnNamedRead read = createNamedRead(select, options);
+ TxnNamedRead read = createNamedRead(select, options, state);
keyConsumer.accept(read.key());
reads.add(read);
}
if (returningSelect != null)
{
- for (TxnNamedRead read : createNamedReads(returningSelect,
options))
+ for (TxnNamedRead read : createNamedReads(returningSelect,
options, state))
{
keyConsumer.accept(read.key());
reads.add(read);
@@ -222,7 +225,7 @@ public class TransactionStatement implements CQLStatement
for (NamedSelect select : autoReads.values())
// don't need keyConsumer as the keys are known to exist due to
Modification
- reads.add(createNamedRead(select, options));
+ reads.add(createNamedRead(select, options, state));
return reads;
}
@@ -284,7 +287,7 @@ public class TransactionStatement implements CQLStatement
{
// TODO: Test case around this...
Preconditions.checkState(conditions.isEmpty(), "No condition
should exist without updates present");
- List<TxnNamedRead> reads = createNamedReads(options, keySet::add);
+ List<TxnNamedRead> reads = createNamedReads(options, state,
keySet::add);
Keys txnKeys = toKeys(keySet);
TxnRead read = new TxnRead(reads, txnKeys);
return new Txn.InMemory(txnKeys, read, TxnQuery.ALL);
@@ -292,7 +295,7 @@ public class TransactionStatement implements CQLStatement
else
{
TxnUpdate update = createUpdate(state, options, keySet::add);
- List<TxnNamedRead> reads = createNamedReads(options, keySet::add);
+ List<TxnNamedRead> reads = createNamedReads(options, state,
keySet::add);
Keys txnKeys = toKeys(keySet);
TxnRead read = new TxnRead(reads, txnKeys);
return new Txn.InMemory(txnKeys, read, TxnQuery.ALL, update);
@@ -301,7 +304,7 @@ public class TransactionStatement implements CQLStatement
private static void checkAtMostOneRowSpecified(ClientState clientState,
@Nullable QueryOptions options, SelectStatement select, String failureMessage)
{
- if (select.getRestrictions().hasAllPKColumnsRestrictedByEqualities())
+ if
(select.getRestrictions().hasAllPrimaryKeyColumnsRestrictedByEqualities())
return;
if (options == null)
@@ -316,7 +319,7 @@ public class TransactionStatement implements CQLStatement
int limit = select.getLimit(options);
QueryOptions finalOptions = options; // javac thinks this is mutable
so requires a copy
- checkTrue(limit == 1, failureMessage, LazyToString.lazy(() ->
select.asCQL(finalOptions, clientState)));
+ checkTrue(limit == 1 &&
select.getRestrictions().hasAllPartitionKeyColumnsRestrictedByEqualities(),
failureMessage, LazyToString.lazy(() -> select.asCQL(finalOptions,
clientState)));
}
@Override
@@ -336,7 +339,11 @@ public class TransactionStatement implements CQLStatement
if (returningSelect != null)
{
- SinglePartitionReadQuery.Group<SinglePartitionReadCommand>
selectQuery = (SinglePartitionReadQuery.Group<SinglePartitionReadCommand>)
returningSelect.select.getQuery(options, 0);
+ ReadQuery readQuery = returningSelect.select.getQuery(options,
0);
+ checkTrue(readQuery instanceof
SinglePartitionReadQuery.Group, ILLEGAL_RANGE_QUERY_MESSAGE,
returningSelect.select.asCQL(options, state.getClientState()));
+
+ @SuppressWarnings("unchecked")
+ SinglePartitionReadQuery.Group<SinglePartitionReadCommand>
selectQuery = (SinglePartitionReadQuery.Group<SinglePartitionReadCommand>)
readQuery;
Selection.Selectors selectors =
returningSelect.select.getSelection().newSelectors(options);
ResultSetBuilder result = new
ResultSetBuilder(returningSelect.select.getResultMetadata(), selectors, null);
if (selectQuery.queries.size() == 1)
diff --git
a/test/unit/org/apache/cassandra/cql3/statements/TransactionStatementTest.java
b/test/unit/org/apache/cassandra/cql3/statements/TransactionStatementTest.java
index 5cb6a2126f..d8062a39ec 100644
---
a/test/unit/org/apache/cassandra/cql3/statements/TransactionStatementTest.java
+++
b/test/unit/org/apache/cassandra/cql3/statements/TransactionStatementTest.java
@@ -53,6 +53,7 @@ public class TransactionStatementTest
private static final TableId TABLE2_ID =
TableId.fromString("00000000-0000-0000-0000-000000000002");
private static final TableId TABLE3_ID =
TableId.fromString("00000000-0000-0000-0000-000000000003");
private static final TableId TABLE4_ID =
TableId.fromString("00000000-0000-0000-0000-000000000004");
+ private static final TableId TABLE5_ID =
TableId.fromString("00000000-0000-0000-0000-000000000005");
@BeforeClass
public static void beforeClass() throws Exception
@@ -62,7 +63,8 @@ public class TransactionStatementTest
parse("CREATE TABLE tbl1 (k int, c int, v
int, primary key (k, c))", "ks").id(TABLE1_ID),
parse("CREATE TABLE tbl2 (k int, c int, v
int, primary key (k, c))", "ks").id(TABLE2_ID),
parse("CREATE TABLE tbl3 (k int PRIMARY
KEY, \"with spaces\" int, \"with\"\"quote\" int, \"MiXeD_CaSe\" int)",
"ks").id(TABLE3_ID),
- parse("CREATE TABLE tbl4 (k int PRIMARY
KEY, int_list list<int>)", "ks").id(TABLE4_ID));
+ parse("CREATE TABLE tbl4 (k int PRIMARY
KEY, int_list list<int>)", "ks").id(TABLE4_ID),
+ parse("CREATE TABLE tbl5 (k int PRIMARY
KEY, v int)", "ks").id(TABLE5_ID));
}
@Test
@@ -330,6 +332,30 @@ public class TransactionStatementTest
.hasMessageContaining(String.format(CANNOT_SET_KEY_WITH_REFERENCE_MESSAGE,
"row0.k", "k"));
}
+ @Test
+ public void shouldRejectNormalSelectWithIncompletePartitionKey()
+ {
+ String query = "BEGIN TRANSACTION\n" +
+ " SELECT k, v FROM ks.tbl5 LIMIT 1;\n" +
+ "COMMIT TRANSACTION;\n";
+
+ Assertions.assertThatThrownBy(() -> prepare(query))
+ .isInstanceOf(InvalidRequestException.class)
+
.hasMessageContaining(String.format(INCOMPLETE_PRIMARY_KEY_SELECT_MESSAGE,
"SELECT v FROM ks.tbl5 LIMIT 1"));
+ }
+
+ @Test
+ public void shouldRejectLetSelectWithIncompletePartitionKey()
+ {
+ String query = "BEGIN TRANSACTION\n" +
+ " LET row1 = (SELECT k, v FROM ks.tbl5 WHERE token(k)
> token(123) LIMIT 1); \n" +
+ " SELECT row1.k, row1.v;\n" +
+ "COMMIT TRANSACTION;\n";
+
+ Assertions.assertThatThrownBy(() -> prepare(query))
+ .isInstanceOf(InvalidRequestException.class)
+
.hasMessageContaining(String.format(INCOMPLETE_PRIMARY_KEY_LET_MESSAGE, "SELECT
v FROM ks.tbl5 WHERE token(k) > 0000007b LIMIT 1"));
+ }
private static CQLStatement prepare(String query)
{
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]