diff --git a/plugin/trino-iceberg/src/test/java/io/trino/plugin/iceberg/TestIcebergLocalConcurrentWrites.java b/plugin/trino-iceberg/src/test/java/io/trino/plugin/iceberg/TestIcebergLocalConcurrentWrites.java index 65253a37868..903fff33da5 100644 --- a/plugin/trino-iceberg/src/test/java/io/trino/plugin/iceberg/TestIcebergLocalConcurrentWrites.java +++ b/plugin/trino-iceberg/src/test/java/io/trino/plugin/iceberg/TestIcebergLocalConcurrentWrites.java @@ -393,13 +393,22 @@ void testConcurrentNonOverlappingUpdate() @Test void testConcurrentOverlappingUpdate() throws Exception + { + testConcurrentOverlappingUpdate(false); + testConcurrentOverlappingUpdate(true); + } + + private void testConcurrentOverlappingUpdate(boolean partitioned) + throws Exception { int threads = 3; CyclicBarrier barrier = new CyclicBarrier(threads); ExecutorService executor = newFixedThreadPool(threads); String tableName = "test_concurrent_overlapping_updates_table_" + randomNameSuffix(); - assertUpdate("CREATE TABLE " + tableName + " (a, part) WITH (partitioning = ARRAY['part']) AS VALUES (1, 10), (11, 20), (21, 30), (31, 40)", 4); + assertUpdate("CREATE TABLE " + tableName + " (a, part) " + + (partitioned ? " WITH (partitioning = ARRAY['part'])" : "") + + " AS VALUES (1, 10), (11, 20), (21, 30), (31, 40)", 4); try { List> futures = IntStream.range(0, threads)