Skip to content

Commit cc328cf

Browse files
committed
Disable skip metadata for CQL4 for broken cases
CQL4 has flaws that allows scenario when after schema change affected prepare statement is not invalidated on client side. As result, data that is read by driver does no match cached metadata and client will fail to deserialize or deserialization will go wrong. More info at: scylladb/scylladb#20860 This commit introduces PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD(advanced.prepared-statements.skip-cql4-metadata-resolve-method) that controls how driver resolves skip metadata flag for CQL4 prepared statements, it can be "smart", "always-on", "always-off". Default is "smart". It makes driver disable skip metadata flag only for wildcard selects and selects that returns udts (including collections and maps)
1 parent ce575f3 commit cc328cf

File tree

11 files changed

+403
-7
lines changed

11 files changed

+403
-7
lines changed

core/src/main/java/com/datastax/dse/driver/internal/core/cql/DseConversions.java

Lines changed: 10 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -39,6 +39,7 @@
3939
import com.datastax.oss.driver.internal.core.ProtocolVersionRegistry;
4040
import com.datastax.oss.driver.internal.core.context.InternalDriverContext;
4141
import com.datastax.oss.driver.internal.core.cql.Conversions;
42+
import com.datastax.oss.driver.internal.core.cql.DefaultPreparedStatement;
4243
import com.datastax.oss.protocol.internal.Message;
4344
import com.datastax.oss.protocol.internal.request.Execute;
4445
import com.datastax.oss.protocol.internal.request.Query;
@@ -115,8 +116,15 @@ public static Message toContinuousPagingMessage(
115116
protocolVersion, DefaultProtocolFeature.UNSET_BOUND_VALUES)) {
116117
Conversions.ensureAllSet(boundStatement);
117118
}
118-
boolean skipMetadata =
119-
boundStatement.getPreparedStatement().getResultSetDefinitions().size() > 0;
119+
120+
boolean skipMetadata;
121+
if (boundStatement.getPreparedStatement() instanceof DefaultPreparedStatement) {
122+
skipMetadata =
123+
((DefaultPreparedStatement) boundStatement.getPreparedStatement()).isSkipMetadata();
124+
} else {
125+
skipMetadata = boundStatement.getPreparedStatement().getResultSetDefinitions().size() > 0;
126+
;
127+
}
120128
DseQueryOptions queryOptions =
121129
new DseQueryOptions(
122130
consistencyCode,
Lines changed: 13 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,13 @@
1+
package com.datastax.oss.driver.api.core;
2+
3+
public enum CQL4SkipMetadataResolveMethod {
4+
// SMART (Default) - Disables the skip metadata flag only for wildcard selects (`SELECT * FROM`)
5+
// and queries
6+
// that return UDTs (including UDT collections and maps containing UDTs).
7+
SMART,
8+
// ENABLED – Enables the `skip metadata` flag, preventing metadata from being sent
9+
ENABLED,
10+
// DISABLED - Disables the `skip metadata` flag, ensuring metadata is included in every RESULT
11+
// frame for bound statement execution.
12+
DISABLED,
13+
}

core/src/main/java/com/datastax/oss/driver/api/core/config/DefaultDriverOption.java

Lines changed: 28 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -689,6 +689,34 @@ public enum DefaultDriverOption implements DriverOption {
689689
* <p>Value-type: boolean
690690
*/
691691
PREPARE_ON_ALL_NODES("advanced.prepared-statements.prepare-on-all-nodes"),
692+
693+
/**
694+
* CQL 4.x has a known issue where prepared statement invalidation may be bypassed on the client
695+
* side. Reference: https://github.com/scylladb/scylladb/issues/20860
696+
*
697+
* <p>When this occurs, the client's metadata can become outdated, leading to various
698+
* deserialization errors.
699+
*
700+
* <p>To mitigate this, the driver can disable the `skip metadata` flag, ensuring the server
701+
* includes metadata with every bound statement RESULT query response.
702+
*
703+
* <p>This setting determines how the driver handles the `skip metadata` flag for CQL 4 prepared
704+
* statements: - **"SMART"** (default) – Disables the flag only for wildcard selects (`SELECT *
705+
* FROM`) and queries that return UDTs (including UDT collections and maps containing UDTs). -
706+
* **"ENABLED"** – Enables the `skip metadata` flag, preventing metadata from being sent. -
707+
* **"DISABLED"** – Disables the `skip metadata` flag, ensuring metadata is included in every
708+
* RESULT frame.
709+
*
710+
* <p>Sending metadata reduces performance on both the driver and server while increasing traffic.
711+
* If you need to use UDTs or wildcard selects, you must either accept the performance impact or
712+
* ensure: 1. No schema alterations are performed on tables or UDTs in use. 2. After any schema
713+
* change, all relevant prepared statements are re-prepared.
714+
*
715+
* <p>Value-type: string
716+
*/
717+
PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD(
718+
"advanced.prepared-statements.skip-cql4-metadata-resolve-method"),
719+
692720
/**
693721
* Whether the driver tries to prepare on new nodes at all.
694722
*

core/src/main/java/com/datastax/oss/driver/api/core/config/OptionsMap.java

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,7 @@
1717
*/
1818
package com.datastax.oss.driver.api.core.config;
1919

20+
import com.datastax.oss.driver.api.core.CQL4SkipMetadataResolveMethod;
2021
import com.datastax.oss.driver.shaded.guava.common.collect.ImmutableList;
2122
import edu.umd.cs.findbugs.annotations.NonNull;
2223
import edu.umd.cs.findbugs.annotations.Nullable;
@@ -258,6 +259,9 @@ protected static void fillWithDriverDefaults(OptionsMap map) {
258259
map.put(TypedDriverOption.REQUEST_TIMEOUT, requestTimeout);
259260
map.put(TypedDriverOption.REQUEST_CONSISTENCY, "LOCAL_ONE");
260261
map.put(TypedDriverOption.REQUEST_PAGE_SIZE, requestPageSize);
262+
map.put(
263+
TypedDriverOption.PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD,
264+
CQL4SkipMetadataResolveMethod.SMART.name());
261265
map.put(TypedDriverOption.REQUEST_SERIAL_CONSISTENCY, "SERIAL");
262266
map.put(TypedDriverOption.REQUEST_DEFAULT_IDEMPOTENCE, false);
263267
map.put(TypedDriverOption.GRAPH_TRAVERSAL_SOURCE, "g");

core/src/main/java/com/datastax/oss/driver/api/core/config/TypedDriverOption.java

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -583,6 +583,10 @@ public String toString() {
583583
/** Whether `Session.prepare` calls should be sent to all nodes in the cluster. */
584584
public static final TypedDriverOption<Boolean> PREPARE_ON_ALL_NODES =
585585
new TypedDriverOption<>(DefaultDriverOption.PREPARE_ON_ALL_NODES, GenericType.BOOLEAN);
586+
/** Method to resolve skip metadata flag for CQL4. */
587+
public static final TypedDriverOption<String> PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD =
588+
new TypedDriverOption<>(
589+
DefaultDriverOption.PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD, GenericType.STRING);
586590
/** Whether the driver tries to prepare on new nodes at all. */
587591
public static final TypedDriverOption<Boolean> REPREPARE_ENABLED =
588592
new TypedDriverOption<>(DefaultDriverOption.REPREPARE_ENABLED, GenericType.BOOLEAN);

core/src/main/java/com/datastax/oss/driver/internal/core/cql/Conversions.java

Lines changed: 17 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -194,8 +194,15 @@ public static Message toMessage(
194194
protocolVersion, DefaultProtocolFeature.UNSET_BOUND_VALUES)) {
195195
ensureAllSet(boundStatement);
196196
}
197-
boolean skipMetadata =
198-
boundStatement.getPreparedStatement().getResultSetDefinitions().size() > 0;
197+
198+
boolean skipMetadata;
199+
if (boundStatement.getPreparedStatement() instanceof DefaultPreparedStatement) {
200+
skipMetadata =
201+
((DefaultPreparedStatement) boundStatement.getPreparedStatement()).isSkipMetadata();
202+
} else {
203+
skipMetadata = boundStatement.getPreparedStatement().getResultSetDefinitions().size() > 0;
204+
}
205+
199206
QueryOptions queryOptions =
200207
new QueryOptions(
201208
consistencyCode,
@@ -382,6 +389,12 @@ public static DefaultPreparedStatement toPreparedStatement(
382389

383390
Partitioner partitioner = PartitionerFactory.partitioner(variableDefinitions, context);
384391

392+
DriverExecutionProfile defaultExecutionProfile =
393+
(request.getExecutionProfileNameForBoundStatements() == null
394+
|| request.getExecutionProfileNameForBoundStatements().isEmpty())
395+
? context.getConfig().getDefaultProfile()
396+
: context.getConfig().getProfile(request.getExecutionProfileNameForBoundStatements());
397+
385398
return new DefaultPreparedStatement(
386399
ByteBuffer.wrap(response.preparedQueryId).asReadOnlyBuffer(),
387400
request.getQuery(),
@@ -409,7 +422,8 @@ public static DefaultPreparedStatement toPreparedStatement(
409422
request.areBoundStatementsTracing(),
410423
context.getCodecRegistry(),
411424
context.getProtocolVersion(),
412-
lwtInfo != null && lwtInfo.isLwt(response.variablesMetadata.flags));
425+
lwtInfo != null && lwtInfo.isLwt(response.variablesMetadata.flags),
426+
defaultExecutionProfile);
413427
}
414428

415429
public static ColumnDefinitions toColumnDefinitions(

core/src/main/java/com/datastax/oss/driver/internal/core/cql/DefaultPreparedStatement.java

Lines changed: 134 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -23,29 +23,41 @@
2323
*/
2424
package com.datastax.oss.driver.internal.core.cql;
2525

26+
import com.datastax.oss.driver.api.core.CQL4SkipMetadataResolveMethod;
2627
import com.datastax.oss.driver.api.core.ConsistencyLevel;
2728
import com.datastax.oss.driver.api.core.CqlIdentifier;
2829
import com.datastax.oss.driver.api.core.ProtocolVersion;
30+
import com.datastax.oss.driver.api.core.config.DefaultDriverOption;
2931
import com.datastax.oss.driver.api.core.config.DriverExecutionProfile;
3032
import com.datastax.oss.driver.api.core.cql.BoundStatement;
3133
import com.datastax.oss.driver.api.core.cql.BoundStatementBuilder;
34+
import com.datastax.oss.driver.api.core.cql.ColumnDefinition;
3235
import com.datastax.oss.driver.api.core.cql.ColumnDefinitions;
3336
import com.datastax.oss.driver.api.core.cql.PreparedStatement;
3437
import com.datastax.oss.driver.api.core.cql.Statement;
3538
import com.datastax.oss.driver.api.core.metadata.token.Partitioner;
3639
import com.datastax.oss.driver.api.core.metadata.token.Token;
40+
import com.datastax.oss.driver.api.core.type.ContainerType;
41+
import com.datastax.oss.driver.api.core.type.DataType;
42+
import com.datastax.oss.driver.api.core.type.MapType;
43+
import com.datastax.oss.driver.api.core.type.TupleType;
44+
import com.datastax.oss.driver.api.core.type.UserDefinedType;
3745
import com.datastax.oss.driver.api.core.type.codec.registry.CodecRegistry;
3846
import com.datastax.oss.driver.internal.core.data.ValuesHelper;
3947
import com.datastax.oss.driver.internal.core.session.RepreparePayload;
48+
import com.datastax.oss.driver.shaded.guava.common.base.Splitter;
4049
import edu.umd.cs.findbugs.annotations.NonNull;
4150
import java.nio.ByteBuffer;
4251
import java.time.Duration;
4352
import java.util.List;
4453
import java.util.Map;
4554
import net.jcip.annotations.ThreadSafe;
55+
import org.slf4j.Logger;
56+
import org.slf4j.LoggerFactory;
4657

4758
@ThreadSafe
4859
public class DefaultPreparedStatement implements PreparedStatement {
60+
private static final Logger LOGGER = LoggerFactory.getLogger(DefaultPreparedStatement.class);
4961

5062
private final ByteBuffer id;
5163
private final RepreparePayload repreparePayload;
@@ -69,6 +81,8 @@ public class DefaultPreparedStatement implements PreparedStatement {
6981
private final Duration timeoutForBoundStatements;
7082
private final Partitioner partitioner;
7183
private final boolean isLWT;
84+
private volatile boolean skipMetadata;
85+
private final DriverExecutionProfile defaultExecutionProfile;
7286

7387
public DefaultPreparedStatement(
7488
ByteBuffer id,
@@ -95,7 +109,8 @@ public DefaultPreparedStatement(
95109
boolean areBoundStatementsTracing,
96110
CodecRegistry codecRegistry,
97111
ProtocolVersion protocolVersion,
98-
boolean isLWT) {
112+
boolean isLWT,
113+
DriverExecutionProfile defaultExecutionProfile) {
99114
this.id = id;
100115
this.partitionKeyIndices = partitionKeyIndices;
101116
// It's important that we keep a reference to this object, so that it only gets evicted from
@@ -122,6 +137,15 @@ public DefaultPreparedStatement(
122137
this.codecRegistry = codecRegistry;
123138
this.protocolVersion = protocolVersion;
124139
this.isLWT = isLWT;
140+
this.defaultExecutionProfile = defaultExecutionProfile;
141+
this.skipMetadata =
142+
resolveSkipMetadata(
143+
query,
144+
resultMetadataId,
145+
resultSetDefinitions,
146+
executionProfileForBoundStatements != null
147+
? executionProfileForBoundStatements
148+
: this.defaultExecutionProfile);
125149
}
126150

127151
@NonNull
@@ -147,6 +171,10 @@ public Partitioner getPartitioner() {
147171
return partitioner;
148172
}
149173

174+
public boolean isSkipMetadata() {
175+
return skipMetadata;
176+
}
177+
150178
@NonNull
151179
@Override
152180
public List<Integer> getPartitionKeyIndices() {
@@ -172,6 +200,15 @@ public boolean isLWT() {
172200
@Override
173201
public void setResultMetadata(
174202
@NonNull ByteBuffer newResultMetadataId, @NonNull ColumnDefinitions newResultSetDefinitions) {
203+
this.skipMetadata =
204+
resolveSkipMetadata(
205+
this.getQuery(),
206+
newResultMetadataId,
207+
newResultSetDefinitions,
208+
executionProfileForBoundStatements != null
209+
? executionProfileForBoundStatements
210+
: this.defaultExecutionProfile);
211+
175212
this.resultMetadata = new ResultMetadata(newResultMetadataId, newResultSetDefinitions);
176213
}
177214

@@ -242,4 +279,100 @@ private ResultMetadata(ByteBuffer resultMetadataId, ColumnDefinitions resultSetD
242279
this.resultSetDefinitions = resultSetDefinitions;
243280
}
244281
}
282+
283+
private static boolean resolveSkipMetadata(
284+
String query,
285+
ByteBuffer resultMetadataId,
286+
ColumnDefinitions resultSet,
287+
DriverExecutionProfile executionProfileForBoundStatements) {
288+
if (resultSet == null || resultSet.size() == 0) {
289+
// there is no reason to send this flag, there will be no rows in the response and,
290+
// consequently, no metadata.
291+
return false;
292+
}
293+
if (resultMetadataId != null && resultMetadataId.capacity() > 0) {
294+
// Result metadata ID feature is supported, it makes prepared statement invalidation work
295+
// properly.
296+
// Skip Metadata should be enabled.
297+
// Prepared statement invalidation works perfectly no need to disable skip metadata
298+
return true;
299+
}
300+
301+
CQL4SkipMetadataResolveMethod resolveMethod = CQL4SkipMetadataResolveMethod.SMART;
302+
303+
if (executionProfileForBoundStatements != null) {
304+
String resolveMethodName =
305+
executionProfileForBoundStatements.getString(
306+
DefaultDriverOption.PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD);
307+
try {
308+
resolveMethod = CQL4SkipMetadataResolveMethod.valueOf(resolveMethodName);
309+
} catch (IllegalArgumentException e) {
310+
LOGGER.warn(
311+
"Property advanced.prepared-statements.skip-cql4-metadata-resolve-method is incorrectly set to `{}`, "
312+
+ "available options: smart, ENABLED, DISABLED. Defaulting to `SMART`",
313+
resolveMethodName);
314+
resolveMethod = CQL4SkipMetadataResolveMethod.SMART;
315+
}
316+
}
317+
318+
switch (resolveMethod) {
319+
case ENABLED:
320+
return true;
321+
case DISABLED:
322+
return false;
323+
}
324+
325+
List<String> chunks = Splitter.onPattern("\\s+").splitToList(query);
326+
if (chunks.size() < 2) {
327+
// Weird query, assuming no result expected
328+
return false;
329+
}
330+
if (!chunks.get(0).toLowerCase().startsWith("select")) {
331+
// In case if non-select sneaks in, disable skip metadata for it no result expected.
332+
return false;
333+
}
334+
if (chunks.get(1).equals("*")) {
335+
LOGGER.warn(
336+
"Prepared statement {} is a wildcard select, which can cause prepared statement invalidation issues when executed on CQL4. "
337+
+ "These issues may lead to broken deserialization or data corruption. "
338+
+ "To mitigate this, the driver ensures that the server returns metadata with each query for such statements, "
339+
+ "though this negatively impacts performance. "
340+
+ "To avoid this, consider using a targeted select instead. "
341+
+ "Find more mitigation options in description of `advanced.prepared-statements.skip-cql4-metadata-resolve-method` flag",
342+
query);
343+
return false;
344+
}
345+
// Disable skipping metadata if results contains udt and
346+
for (ColumnDefinition columnDefinition : resultSet) {
347+
if (containsUDT(columnDefinition.getType())) {
348+
LOGGER.warn(
349+
"Prepared statement {} contains UDT in result, which can cause prepared statement invalidation issues when executed on CQL4. "
350+
+ "These issues may lead to broken deserialization or data corruption. "
351+
+ "To mitigate this, the driver ensures that the server returns metadata with each query for such statements, "
352+
+ "though this negatively impacts performance. "
353+
+ "To avoid this, consider using regular columns instead of UDT. "
354+
+ "Find more mitigation options in description of `advanced.prepared-statements.skip-cql4-metadata-resolve-method` flag",
355+
query);
356+
return false;
357+
}
358+
}
359+
return true;
360+
}
361+
362+
private static boolean containsUDT(DataType dataType) {
363+
if (dataType instanceof ContainerType) {
364+
return containsUDT(((ContainerType) dataType).getElementType());
365+
} else if (dataType instanceof TupleType) {
366+
for (DataType elementType : ((TupleType) dataType).getComponentTypes()) {
367+
if (containsUDT(elementType)) {
368+
return true;
369+
}
370+
}
371+
return false;
372+
} else if (dataType instanceof MapType) {
373+
return containsUDT(((MapType) dataType).getKeyType())
374+
|| containsUDT(((MapType) dataType).getValueType());
375+
}
376+
return dataType instanceof UserDefinedType;
377+
}
245378
}

core/src/main/resources/reference.conf

Lines changed: 21 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -2182,6 +2182,27 @@ datastax-java-driver {
21822182
# Overridable in a profile: yes
21832183
prepare-on-all-nodes = true
21842184

2185+
# CQL 4.x has a known issue where prepared statement invalidation may be bypassed on the client side.
2186+
# Reference: https://github.com/scylladb/scylladb/issues/20860
2187+
# When this occurs, the client's metadata can become outdated, leading to various deserialization errors.
2188+
#
2189+
# To mitigate this, the driver can disable the `skip metadata` flag, ensuring the server includes metadata with every bound statement RESULT query response.
2190+
# This setting determines how the driver handles the `skip metadata` flag for CQL 4 prepared statements:
2191+
# - **"SMART"** (default) – Disables the flag only for wildcard selects (`SELECT * FROM`) and queries
2192+
# that return UDTs (including UDT collections and maps containing UDTs).
2193+
# - **"ENABLED"** – Enables the `skip metadata` flag, preventing metadata from being sent.
2194+
# - **"DISABLED"** – Disables the `skip metadata` flag, ensuring metadata is included in every RESULT frame.
2195+
#
2196+
# Sending metadata reduces performance on both the driver and server while increasing traffic.
2197+
# If you need to use UDTs or wildcard selects, you must either accept the performance impact or ensure:
2198+
# 1. No schema alterations are performed on tables or UDTs in use.
2199+
# 2. After any schema change, all relevant prepared statements are re-prepared.
2200+
#
2201+
# Required: yes
2202+
# Modifiable at runtime: yes, the new value will be used for requests issued after the change.
2203+
# Overridable in a profile: yes
2204+
skip-cql4-metadata-resolve-method = smart
2205+
21852206
# How the driver replicates prepared statements on a node that just came back up or joined the
21862207
# cluster.
21872208
reprepare-on-up {

core/src/test/java/com/datastax/oss/driver/internal/core/cql/RequestHandlerTestHarness.java

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -29,6 +29,7 @@
2929
import static org.mockito.Mockito.mock;
3030
import static org.mockito.Mockito.when;
3131

32+
import com.datastax.oss.driver.api.core.CQL4SkipMetadataResolveMethod;
3233
import com.datastax.oss.driver.api.core.CqlIdentifier;
3334
import com.datastax.oss.driver.api.core.DefaultConsistencyLevel;
3435
import com.datastax.oss.driver.api.core.ProtocolVersion;
@@ -122,6 +123,8 @@ protected RequestHandlerTestHarness(Builder builder) {
122123
when(defaultProfile.getBoolean(DefaultDriverOption.REQUEST_DEFAULT_IDEMPOTENCE))
123124
.thenReturn(builder.defaultIdempotence);
124125
when(defaultProfile.getBoolean(DefaultDriverOption.PREPARE_ON_ALL_NODES)).thenReturn(true);
126+
when(defaultProfile.getString(DefaultDriverOption.PREPARE_SKIP_CQL4_METADATA_RESOLVE_METHOD))
127+
.thenReturn(CQL4SkipMetadataResolveMethod.SMART.name());
125128

126129
when(config.getDefaultProfile()).thenReturn(defaultProfile);
127130
when(context.getConfig()).thenReturn(config);

0 commit comments

Comments
 (0)