This is an automated email from the ASF dual-hosted git repository.
adelapena pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git
The following commit(s) were added to refs/heads/trunk by this push:
new b0aa44b27d Add guardrail for partition size and deprecate
compaction_large_partition_warning_threshold
b0aa44b27d is described below
commit b0aa44b27da97b37345ee6fafbee16d66f3b384f
Author: Andrés de la Peña <[email protected]>
AuthorDate: Tue May 9 12:07:29 2023 +0100
Add guardrail for partition size and deprecate
compaction_large_partition_warning_threshold
patch by Andrés de la Peña; reviewed by Berenguer Blasi and Maxwell-Guo for
CASSANDRA-18500
---
CHANGES.txt | 1 +
NEWS.txt | 4 +
conf/cassandra.yaml | 12 +-
src/java/org/apache/cassandra/config/Config.java | 3 +
.../cassandra/config/DatabaseDescriptor.java | 6 +-
.../apache/cassandra/config/GuardrailsOptions.java | 28 ++++
.../apache/cassandra/db/guardrails/Guardrails.java | 34 ++++-
.../cassandra/db/guardrails/GuardrailsConfig.java | 12 ++
.../cassandra/db/guardrails/GuardrailsMBean.java | 27 ++++
.../io/sstable/format/SortedTableWriter.java | 14 ++
.../guardrails/GuardrailPartitionSizeTest.java | 154 +++++++++++++++++++++
.../db/guardrails/GuardrailPartitionSizeTest.java | 123 ++++++++++++++++
12 files changed, 415 insertions(+), 3 deletions(-)
diff --git a/CHANGES.txt b/CHANGES.txt
index 0e3834c05f..168e5b2bed 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
5.0
+ * Add guardrail for partition size and deprecate
compaction_large_partition_warning_threshold (CASSANDRA-18500)
* Add HISTORY command for CQLSH (CASSANDRA-15046)
* Fix sstable formats configuration (CASSANDRA-18441)
* Add guardrail to bound timestamps (CASSANDRA-18352)
diff --git a/NEWS.txt b/NEWS.txt
index 4e0792aee6..5c3267e1e0 100644
--- a/NEWS.txt
+++ b/NEWS.txt
@@ -78,6 +78,7 @@ New features
- Maximum replication factor
- Whether DROP KEYSPACE commands are allowed.
- Column value size
+ - Partition size
- It is possible to list ephemeral snapshots by nodetool listsnaphots
command when flag "-e" is specified.
- Added a new flag to `nodetool profileload` and JMX endpoint to set up
recurring profile load generation on specified
intervals (see CASSANDRA-17821)
@@ -183,6 +184,9 @@ Deprecation
- All native CQL functions names that don't use the snake case names are
deprecated in favour of equivalent names
using snake casing. Thus, `totimestamp` is deprecated in favour of
`to_timestamp`, `intasblob` in favour
of `int_as_blob`, `castAsInt` in favour of `cast_as_int`, etc.
+ - The config property `compaction_large_partition_warning_threshold` has
been deprecated in favour of the new
+ guardrail for partition size. That guardrail is based on the properties
`partition_size_warn_threshold` and
+ `partition_size_fail_threshold`. The warn threshold has a very similar
behaviour to the old config property.
4.1
===
diff --git a/conf/cassandra.yaml b/conf/cassandra.yaml
index 72fe17fa63..322b84a6cf 100644
--- a/conf/cassandra.yaml
+++ b/conf/cassandra.yaml
@@ -1541,7 +1541,8 @@ batch_size_fail_threshold: 50KiB
# Log WARN on any batches not of type LOGGED than span across more partitions
than this limit
unlogged_batch_across_partitions_warn_threshold: 10
-# Log a warning when compacting partitions larger than this value
+# Log a warning when compacting partitions larger than this value.
+# As of Cassandra 5.0, this property is deprecated in favour of
partition_size_warn_threshold.
compaction_large_partition_warning_threshold: 100MiB
# Log a warning when writing more tombstones than this value to a partition
@@ -1813,6 +1814,15 @@ drop_compact_storage_enabled: false
# write_consistency_levels_warned: []
# write_consistency_levels_disallowed: []
#
+# Guardrail to warn or fail when writing partitions larger than threshold,
expressed as 100MiB, 1GiB, etc.
+# The guardrail is only checked when writing sstables (flush and compaction),
and exceeding the fail threshold on that
+# moment will only log an error message, without interrupting the operation.
+# This operates on a per-sstable basis, so it won't detect a large partition
if it is spread across multiple sstables.
+# The warning threshold replaces the deprecated config property
compaction_large_partition_warning_threshold.
+# The two thresholds default to null to disable.
+# partition_size_warn_threshold:
+# partition_size_fail_threshold:
+#
# Guardrail to warn or fail when writing column values larger than threshold.
# This guardrail is only applied to the values of regular columns because both
the serialized partitions keys and the
# values of the components of the clustering key already have a fixed,
relatively small size limit of 65535 bytes, which
diff --git a/src/java/org/apache/cassandra/config/Config.java
b/src/java/org/apache/cassandra/config/Config.java
index fa83f51421..7b87d61d0d 100644
--- a/src/java/org/apache/cassandra/config/Config.java
+++ b/src/java/org/apache/cassandra/config/Config.java
@@ -327,6 +327,7 @@ public class Config
public volatile Integer concurrent_compactors;
@Replaces(oldName = "compaction_throughput_mb_per_sec", converter =
Converters.MEBIBYTES_PER_SECOND_DATA_RATE, deprecated = true)
public volatile DataRateSpec.LongBytesPerSecondBound compaction_throughput
= new DataRateSpec.LongBytesPerSecondBound("64MiB/s");
+ @Deprecated
@Replaces(oldName = "compaction_large_partition_warning_threshold_mb",
converter = Converters.MEBIBYTES_DATA_STORAGE_INT, deprecated = true)
public volatile DataStorageSpec.IntMebibytesBound
compaction_large_partition_warning_threshold = new
DataStorageSpec.IntMebibytesBound("100MiB");
@Replaces(oldName = "min_free_space_per_drive_in_mb", converter =
Converters.MEBIBYTES_DATA_STORAGE_INT, deprecated = true)
@@ -872,6 +873,8 @@ public class Config
public volatile boolean read_before_write_list_operations_enabled = true;
public volatile boolean allow_filtering_enabled = true;
public volatile boolean simplestrategy_enabled = true;
+ public volatile DataStorageSpec.LongBytesBound
partition_size_warn_threshold = null;
+ public volatile DataStorageSpec.LongBytesBound
partition_size_fail_threshold = null;
public volatile DataStorageSpec.LongBytesBound
column_value_size_warn_threshold = null;
public volatile DataStorageSpec.LongBytesBound
column_value_size_fail_threshold = null;
public volatile DataStorageSpec.LongBytesBound
collection_size_warn_threshold = null;
diff --git a/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
b/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
index c15de77b95..2f56d73de5 100644
--- a/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
+++ b/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
@@ -2171,7 +2171,11 @@ public class DatabaseDescriptor
conf.compaction_throughput = new
DataRateSpec.LongBytesPerSecondBound(value, MEBIBYTES_PER_SECOND);
}
- public static long getCompactionLargePartitionWarningThreshold() { return
conf.compaction_large_partition_warning_threshold.toBytesInLong(); }
+ @Deprecated
+ public static long getCompactionLargePartitionWarningThreshold()
+ {
+ return
conf.compaction_large_partition_warning_threshold.toBytesInLong();
+ }
public static int getCompactionTombstoneWarningThreshold()
{
diff --git a/src/java/org/apache/cassandra/config/GuardrailsOptions.java
b/src/java/org/apache/cassandra/config/GuardrailsOptions.java
index 9734cb7cd2..7affe6f7cf 100644
--- a/src/java/org/apache/cassandra/config/GuardrailsOptions.java
+++ b/src/java/org/apache/cassandra/config/GuardrailsOptions.java
@@ -76,6 +76,7 @@ public class GuardrailsOptions implements GuardrailsConfig
config.read_consistency_levels_disallowed =
validateConsistencyLevels(config.read_consistency_levels_disallowed,
"read_consistency_levels_disallowed");
config.write_consistency_levels_warned =
validateConsistencyLevels(config.write_consistency_levels_warned,
"write_consistency_levels_warned");
config.write_consistency_levels_disallowed =
validateConsistencyLevels(config.write_consistency_levels_disallowed,
"write_consistency_levels_disallowed");
+ validateSizeThreshold(config.partition_size_warn_threshold,
config.partition_size_fail_threshold, false, "partition_size");
validateSizeThreshold(config.column_value_size_warn_threshold,
config.column_value_size_fail_threshold, false, "column_value_size");
validateSizeThreshold(config.collection_size_warn_threshold,
config.collection_size_fail_threshold, false, "collection_size");
validateMaxIntThreshold(config.items_per_collection_warn_threshold,
config.items_per_collection_fail_threshold, "items_per_collection");
@@ -539,6 +540,33 @@ public class GuardrailsOptions implements GuardrailsConfig
x ->
config.write_consistency_levels_disallowed = x);
}
+ @Override
+ @Nullable
+ public DataStorageSpec.LongBytesBound getPartitionSizeWarnThreshold()
+ {
+ return config.partition_size_warn_threshold;
+ }
+
+ @Override
+ @Nullable
+ public DataStorageSpec.LongBytesBound getPartitionSizeFailThreshold()
+ {
+ return config.partition_size_fail_threshold;
+ }
+
+ public void setPartitionSizeThreshold(@Nullable
DataStorageSpec.LongBytesBound warn, @Nullable DataStorageSpec.LongBytesBound
fail)
+ {
+ validateSizeThreshold(warn, fail, false, "partition_size");
+ updatePropertyWithLogging("partition_size_warn_threshold",
+ warn,
+ () -> config.partition_size_warn_threshold,
+ x -> config.partition_size_warn_threshold =
x);
+ updatePropertyWithLogging("partition_size_fail_threshold",
+ fail,
+ () -> config.partition_size_fail_threshold,
+ x -> config.partition_size_fail_threshold =
x);
+ }
+
@Override
@Nullable
public DataStorageSpec.LongBytesBound getColumnValueSizeWarnThreshold()
diff --git a/src/java/org/apache/cassandra/db/guardrails/Guardrails.java
b/src/java/org/apache/cassandra/db/guardrails/Guardrails.java
index 18f9cf345f..d2289b4bb6 100644
--- a/src/java/org/apache/cassandra/db/guardrails/Guardrails.java
+++ b/src/java/org/apache/cassandra/db/guardrails/Guardrails.java
@@ -53,7 +53,7 @@ public final class Guardrails implements GuardrailsMBean
private static final GuardrailsOptions DEFAULT_CONFIG =
DatabaseDescriptor.getGuardrailsConfig();
@VisibleForTesting
- static final Guardrails instance = new Guardrails();
+ public static final Guardrails instance = new Guardrails();
/**
* Guardrail on the total number of user keyspaces.
@@ -311,6 +311,18 @@ public final class Guardrails implements GuardrailsMBean
state ->
CONFIG_PROVIDER.getOrCreate(state).getWriteConsistencyLevelsDisallowed(),
"write consistency levels");
+ /**
+ * Guardrail on the size of a partition.
+ */
+ public static final MaxThreshold partitionSize =
+ new MaxThreshold("partition_size",
+ "Too large partitions can cause performance problems. ",
+ state ->
sizeToBytes(CONFIG_PROVIDER.getOrCreate(state).getPartitionSizeWarnThreshold()),
+ state ->
sizeToBytes(CONFIG_PROVIDER.getOrCreate(state).getPartitionSizeFailThreshold()),
+ (isWarning, what, value, threshold) ->
+ format("Partition %s has size %s, this exceeds
the %s threshold of %s.",
+ what, value, isWarning ? "warning" :
"failure", threshold));
+
/**
* Guardrail on the size of a collection.
*/
@@ -791,6 +803,26 @@ public final class Guardrails implements GuardrailsMBean
DEFAULT_CONFIG.setPartitionKeysInSelectThreshold(warn, fail);
}
+ @Override
+ @Nullable
+ public String getPartitionSizeWarnThreshold()
+ {
+ return sizeToString(DEFAULT_CONFIG.getPartitionSizeWarnThreshold());
+ }
+
+ @Override
+ @Nullable
+ public String getPartitionSizeFailThreshold()
+ {
+ return sizeToString(DEFAULT_CONFIG.getPartitionSizeFailThreshold());
+ }
+
+ @Override
+ public void setPartitionSizeThreshold(@Nullable String warnSize, @Nullable
String failSize)
+ {
+ DEFAULT_CONFIG.setPartitionSizeThreshold(sizeFromString(warnSize),
sizeFromString(failSize));
+ }
+
@Override
@Nullable
public String getColumnValueSizeWarnThreshold()
diff --git a/src/java/org/apache/cassandra/db/guardrails/GuardrailsConfig.java
b/src/java/org/apache/cassandra/db/guardrails/GuardrailsConfig.java
index 12d24028e5..af1b5b48c0 100644
--- a/src/java/org/apache/cassandra/db/guardrails/GuardrailsConfig.java
+++ b/src/java/org/apache/cassandra/db/guardrails/GuardrailsConfig.java
@@ -238,6 +238,18 @@ public interface GuardrailsConfig
*/
Set<ConsistencyLevel> getWriteConsistencyLevelsDisallowed();
+ /**
+ * @return The threshold to warn when writing partitions larger than
threshold.
+ */
+ @Nullable
+ DataStorageSpec.LongBytesBound getPartitionSizeWarnThreshold();
+
+ /**
+ * @return The threshold to fail when writing partitions larger than
threshold.
+ */
+ @Nullable
+ DataStorageSpec.LongBytesBound getPartitionSizeFailThreshold();
+
/**
* @return The threshold to warn when writing column values larger than
threshold.
*/
diff --git a/src/java/org/apache/cassandra/db/guardrails/GuardrailsMBean.java
b/src/java/org/apache/cassandra/db/guardrails/GuardrailsMBean.java
index 5f66d71d8d..cabbe0c59e 100644
--- a/src/java/org/apache/cassandra/db/guardrails/GuardrailsMBean.java
+++ b/src/java/org/apache/cassandra/db/guardrails/GuardrailsMBean.java
@@ -468,6 +468,33 @@ public interface GuardrailsMBean
*/
void setWriteConsistencyLevelsDisallowedCSV(String consistencyLevels);
+ /**
+ * @return The threshold to warn when encountering partitions larger than
threshold, as a string formatted as in,
+ * for example, {@code 10GiB}, {@code 20MiB}, {@code 30KiB} or {@code
40B}. A {@code null} value means disabled.
+ */
+ @Nullable
+ String getPartitionSizeWarnThreshold();
+
+ /**
+ * @return The threshold to fail when encountering partitions larger than
threshold, as a string formatted as in,
+ * for example, {@code 10GiB}, {@code 20MiB}, {@code 30KiB} or {@code
40B}. A {@code null} value means disabled.
+ * Triggering a failure emits a log message and a diagnostic event, but
it doesn't throw an exception interrupting
+ * the offending sstable write.
+ */
+ @Nullable
+ String getPartitionSizeFailThreshold();
+
+ /**
+ * @param warnSize The threshold to warn when encountering partitions
larger than threshold, as a string formatted
+ * as in, for example, {@code 10GiB}, {@code 20MiB},
{@code 30KiB} or {@code 40B}.
+ * A {@code null} value means disabled.
+ * @param failSize The threshold to fail when encountering partitions
larger than threshold, as a string formatted
+ * as in, for example, {@code 10GiB}, {@code 20MiB},
{@code 30KiB} or {@code 40B}.
+ * A {@code null} value means disabled. Triggering a
failure emits a log message and a diagnostic
+ * event, but it desn't throw an exception interrupting
the offending sstable write.
+ */
+ void setPartitionSizeThreshold(@Nullable String warnSize, @Nullable String
failSize);
+
/**
* @return The threshold to warn when encountering column values larger
than threshold, as a string formatted as
* in, for example, {@code 10GiB}, {@code 20MiB}, {@code 30KiB} or {@code
40B}. A {@code null} value means disabled.
diff --git
a/src/java/org/apache/cassandra/io/sstable/format/SortedTableWriter.java
b/src/java/org/apache/cassandra/io/sstable/format/SortedTableWriter.java
index d8e512805f..377169faf0 100644
--- a/src/java/org/apache/cassandra/io/sstable/format/SortedTableWriter.java
+++ b/src/java/org/apache/cassandra/io/sstable/format/SortedTableWriter.java
@@ -213,6 +213,7 @@ public abstract class SortedTableWriter<P extends
SortedTablePartitionWriter> ex
long endPosition = dataWriter.position();
long rowSize = endPosition - partitionWriter.getInitialPosition();
+ guardPartitionSize(key, rowSize);
maybeLogLargePartitionWarning(key, rowSize);
maybeLogManyTombstonesWarning(key, metadataCollector.totalTombstones);
metadataCollector.addPartitionSizeInBytes(rowSize);
@@ -324,6 +325,19 @@ public abstract class SortedTableWriter<P extends
SortedTablePartitionWriter> ex
return dataFile;
}
+ private void guardPartitionSize(DecoratedKey key, long rowSize)
+ {
+ if (Guardrails.partitionSize.triggersOn(rowSize, null))
+ {
+ String what = String.format("%s.%s:%s on sstable %s",
+ metadata.keyspace,
+ metadata.name,
+
metadata().partitionKeyType.getString(key.getKey()),
+ getFilename());
+ Guardrails.partitionSize.guard(rowSize, what, true, null);
+ }
+ }
+
private void maybeLogLargePartitionWarning(DecoratedKey key, long rowSize)
{
if (rowSize >
DatabaseDescriptor.getCompactionLargePartitionWarningThreshold())
diff --git
a/test/distributed/org/apache/cassandra/distributed/test/guardrails/GuardrailPartitionSizeTest.java
b/test/distributed/org/apache/cassandra/distributed/test/guardrails/GuardrailPartitionSizeTest.java
new file mode 100644
index 0000000000..b2ee8d7d94
--- /dev/null
+++
b/test/distributed/org/apache/cassandra/distributed/test/guardrails/GuardrailPartitionSizeTest.java
@@ -0,0 +1,154 @@
+/*
+ * 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.cassandra.distributed.test.guardrails;
+
+import java.io.IOException;
+
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.Test;
+
+import org.apache.cassandra.db.guardrails.Guardrails;
+import org.apache.cassandra.distributed.Cluster;
+import org.apache.cassandra.distributed.api.ConsistencyLevel;
+
+import static java.nio.ByteBuffer.allocate;
+
+/**
+ * Tests the guardrail for the size of partition size, {@link
Guardrails#partitionSize}.
+ * <p>
+ * This test only includes the activation of the guardrail during sstable
writes, focusing on the emmitted log messages.
+ * The tests for config, client warnings and diagnostic events are in
+ * {@link org.apache.cassandra.db.guardrails.GuardrailPartitionSizeTest}.
+ */
+public class GuardrailPartitionSizeTest extends GuardrailTester
+{
+ private static final int WARN_THRESHOLD = 1024 * 1024; // bytes (1 MiB)
+ private static final int FAIL_THRESHOLD = WARN_THRESHOLD * 2; // bytes (2
MiB)
+
+ private static final int NUM_NODES = 1;
+ private static final int NUM_CLUSTERINGS = 5;
+
+ private static Cluster cluster;
+
+ @BeforeClass
+ public static void setupCluster() throws IOException
+ {
+ cluster = init(Cluster.build(NUM_NODES)
+ .withConfig(c ->
c.set("partition_size_warn_threshold", WARN_THRESHOLD + "B")
+
.set("partition_size_fail_threshold", FAIL_THRESHOLD + "B")
+
.set("compaction_large_partition_warning_threshold", "999GiB")
+ .set("memtable_heap_space",
"512MiB")) // avoids flushes
+ .start());
+ cluster.disableAutoCompaction(KEYSPACE);
+ }
+
+ @AfterClass
+ public static void teardownCluster()
+ {
+ if (cluster != null)
+ cluster.close();
+ }
+
+ @Override
+ protected Cluster getCluster()
+ {
+ return cluster;
+ }
+
+ @Test
+ public void testPartitionSize()
+ {
+ testPartitionSize(WARN_THRESHOLD, FAIL_THRESHOLD);
+ }
+
+ @Test
+ public void testPartitionSizeWithDynamicUpdate()
+ {
+ int warn = WARN_THRESHOLD * 2;
+ int fail = FAIL_THRESHOLD * 2;
+ cluster.get(1).runOnInstance(() ->
Guardrails.instance.setPartitionSizeThreshold(warn + "B", fail + "B"));
+ testPartitionSize(warn, fail);
+ }
+
+ private void testPartitionSize(int warn, int fail)
+ {
+ schemaChange("CREATE TABLE %s (k int, c int, v blob, PRIMARY KEY (k,
c))");
+
+ // empty table
+ assertNotWarnedOnFlush();
+ assertNotWarnedOnCompact();
+
+ // keep partition size lower than thresholds
+ execute("INSERT INTO %s (k, c, v) VALUES (1, 1, ?)", allocate(1));
+ assertNotWarnedOnFlush();
+ assertNotWarnedOnCompact();
+
+ // exceed warn threshold
+ for (int c = 0; c < NUM_CLUSTERINGS; c++)
+ {
+ execute("INSERT INTO %s (k, c, v) VALUES (1, ?, ?)", c,
allocate(warn / NUM_CLUSTERINGS));
+ }
+ assertWarnedOnFlush(expectedMessages(1));
+ assertWarnedOnCompact(expectedMessages(1));
+
+ // exceed fail threshold
+ for (int c = 0; c < NUM_CLUSTERINGS * 10; c++)
+ {
+ execute("INSERT INTO %s (k, c, v) VALUES (1, ?, ?)", c,
allocate(fail / NUM_CLUSTERINGS));
+ }
+ assertFailedOnFlush(expectedMessages(1));
+ assertFailedOnCompact(expectedMessages(1));
+
+ // remove most of the data to be under the threshold again
+ execute("DELETE FROM %s WHERE k = 1 AND c > 1");
+ assertNotWarnedOnFlush();
+ assertNotWarnedOnCompact();
+
+ // exceed warn threshold in multiple partitions
+ for (int c = 0; c < NUM_CLUSTERINGS; c++)
+ {
+ execute("INSERT INTO %s (k, c, v) VALUES (1, ?, ?)", c,
allocate(warn / NUM_CLUSTERINGS));
+ execute("INSERT INTO %s (k, c, v) VALUES (2, ?, ?)", c,
allocate(warn / NUM_CLUSTERINGS));
+ }
+ assertWarnedOnFlush(expectedMessages(1, 2));
+ assertWarnedOnCompact(expectedMessages(1, 2));
+
+ // exceed warn threshold in a new partition
+ for (int c = 0; c < NUM_CLUSTERINGS; c++)
+ {
+ execute("INSERT INTO %s (k, c, v) VALUES (3, ?, ?)", c,
allocate(warn / NUM_CLUSTERINGS));
+ }
+ assertWarnedOnFlush(expectedMessages(3));
+ assertWarnedOnCompact(expectedMessages(1, 2, 3));
+ }
+
+ private void execute(String query, Object... args)
+ {
+ cluster.coordinator(1).execute(format(query), ConsistencyLevel.ALL,
args);
+ }
+
+ private String[] expectedMessages(int... keys)
+ {
+ String[] messages = new String[keys.length];
+ for (int i = 0; i < keys.length; i++)
+ messages[i] = String.format("Guardrail partition_size violated:
Partition %s:%d", qualifiedTableName, keys[i]);
+ return messages;
+ }
+}
diff --git
a/test/unit/org/apache/cassandra/db/guardrails/GuardrailPartitionSizeTest.java
b/test/unit/org/apache/cassandra/db/guardrails/GuardrailPartitionSizeTest.java
new file mode 100644
index 0000000000..cf48390ceb
--- /dev/null
+++
b/test/unit/org/apache/cassandra/db/guardrails/GuardrailPartitionSizeTest.java
@@ -0,0 +1,123 @@
+/*
+ * 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.cassandra.db.guardrails;
+
+import java.util.Arrays;
+
+import org.junit.Test;
+
+import org.apache.cassandra.config.DataStorageSpec;
+import org.apache.cassandra.db.marshal.Int32Type;
+
+import static java.nio.ByteBuffer.allocate;
+import static
org.apache.cassandra.config.DataStorageSpec.DataStorageUnit.BYTES;
+
+/**
+ * Tests the guardrail for the size of partitions, {@link
Guardrails#partitionSize}.
+ * <p>
+ * The emission on unredacted log messages is tested in {@link
org.apache.cassandra.distributed.test.guardrails.GuardrailPartitionSizeTest}.
+ */
+public class GuardrailPartitionSizeTest extends ThresholdTester
+{
+ private static final int WARN_THRESHOLD = 1024 * 1024; // bytes (1 MiB)
+ private static final int FAIL_THRESHOLD = WARN_THRESHOLD * 2; // bytes (2
MiB)
+ private static final int NUM_CLUSTERINGS = 10;
+ private static final String REDACTED_MESSAGE = "Guardrail partition_size
violated: Partition <redacted> has size";
+
+ public GuardrailPartitionSizeTest()
+ {
+ super(WARN_THRESHOLD + "B",
+ FAIL_THRESHOLD + "B",
+ Guardrails.partitionSize,
+ Guardrails::setPartitionSizeThreshold,
+ Guardrails::getPartitionSizeWarnThreshold,
+ Guardrails::getPartitionSizeFailThreshold,
+ bytes -> new DataStorageSpec.LongBytesBound(bytes,
BYTES).toString(),
+ size -> new DataStorageSpec.LongBytesBound(size).toBytes());
+ }
+
+ @Test
+ public void testPartitionSizeEnabled() throws Throwable
+ {
+ // Insert enough data to trigger the warning guardrail, but not the
failure one.
+ populateTable(WARN_THRESHOLD);
+
+ flush();
+ listener.assertWarned(REDACTED_MESSAGE);
+ listener.clear();
+
+ compact();
+ listener.assertWarned(REDACTED_MESSAGE);
+ listener.clear();
+
+ // Insert enough data to trigger the failure guardrail.
+ populateTable(FAIL_THRESHOLD);
+
+ flush();
+ listener.assertFailed(REDACTED_MESSAGE);
+ listener.clear();
+
+ compact();
+ listener.assertFailed(REDACTED_MESSAGE);
+ listener.clear();
+
+ // remove most of the data to be under the threshold again
+ assertValid("DELETE FROM %s WHERE k = 1 AND c > 0");
+
+ flush();
+ listener.assertNotWarned();
+ listener.assertNotFailed();
+
+ compact();
+ listener.assertNotWarned();
+ listener.assertNotFailed();
+ listener.clear();
+ }
+
+ @Test
+ public void testPartitionSizeDisabled() throws Throwable
+ {
+ guardrails().setPartitionSizeThreshold(null, null);
+
+ populateTable(FAIL_THRESHOLD);
+
+ flush();
+ listener.assertNotWarned();
+ listener.assertNotFailed();
+
+ compact();
+ listener.assertNotWarned();
+ listener.assertNotFailed();
+ }
+
+ private void populateTable(int threshold) throws Throwable
+ {
+ createTable("CREATE TABLE IF NOT EXISTS %s (k int, c int, v blob,
PRIMARY KEY(k, c))");
+ disableCompaction();
+
+ for (int i = 0; i < NUM_CLUSTERINGS; i++)
+ {
+ final int c = i;
+ assertValid(() -> execute(userClientState,
+ "INSERT INTO %s (k, c, v) VALUES (1, ?,
?)",
+
Arrays.asList(Int32Type.instance.decompose(c),
+ allocate(threshold /
NUM_CLUSTERINGS))));
+ }
+ }
+}
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]