Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -403,6 +403,7 @@
import static org.apache.iceberg.TableProperties.PARQUET_BLOOM_FILTER_COLUMN_ENABLED_PREFIX;
import static org.apache.iceberg.TableProperties.WRITE_DATA_LOCATION;
import static org.apache.iceberg.TableProperties.WRITE_LOCATION_PROVIDER_IMPL;
import static org.apache.iceberg.TableUtil.formatVersion;
import static org.apache.iceberg.expressions.Expressions.alwaysTrue;
import static org.apache.iceberg.types.TypeUtil.indexParents;
import static org.apache.iceberg.util.LocationUtil.stripTrailingSlash;
Expand Down Expand Up @@ -629,7 +630,7 @@ private IcebergTableHandle tableHandleForSnapshot(
tableSnapshotId,
SchemaParser.toJson(tableSchema),
partitionSpec.map(PartitionSpecParser::toJson),
table.operations().current().formatVersion(),
formatVersion(table),
TupleDomain.all(),
TupleDomain.all(),
OptionalLong.empty(),
Expand Down Expand Up @@ -1922,7 +1923,7 @@ private BeginTableExecuteResult<ConnectorTableExecuteHandle, ConnectorTableHandl

validateNotModifyingOldSnapshot(table, icebergTable);

int tableFormatVersion = icebergTable.operations().current().formatVersion();
int tableFormatVersion = formatVersion(icebergTable);
if (tableFormatVersion > OPTIMIZE_MAX_SUPPORTED_TABLE_VERSION) {
throw new TrinoException(NOT_SUPPORTED, format(
"%s is not supported for Iceberg table format version > %d. Table %s format version is %s.",
Expand Down Expand Up @@ -2197,7 +2198,7 @@ private static void validateTableExecuteParameters(
String minRetentionParameterName,
String sessionMinRetentionParameterName)
{
int tableFormatVersion = table.operations().current().formatVersion();
int tableFormatVersion = formatVersion(table);
if (tableFormatVersion > CLEANING_UP_PROCEDURES_MAX_SUPPORTED_TABLE_VERSION) {
// It is not known if future version won't bring any new kind of metadata or data files
// because of the way procedures are implemented it is safer to fail here than to potentially remove
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -191,6 +191,7 @@
import static org.apache.iceberg.TableProperties.PARQUET_COMPRESSION;
import static org.apache.iceberg.TableProperties.WRITE_DATA_LOCATION;
import static org.apache.iceberg.TableProperties.WRITE_LOCATION_PROVIDER_IMPL;
import static org.apache.iceberg.TableUtil.formatVersion;
import static org.apache.iceberg.types.Type.TypeID.BINARY;
import static org.apache.iceberg.types.Type.TypeID.FIXED;
import static org.apache.iceberg.util.LocationUtil.stripTrailingSlash;
Expand Down Expand Up @@ -326,7 +327,7 @@ public static Map<String, Object> getIcebergTableProperties(BaseTable icebergTab
properties.put(LOCATION_PROPERTY, icebergTable.location());
}

int formatVersion = icebergTable.operations().current().formatVersion();
int formatVersion = formatVersion(icebergTable);
properties.put(FORMAT_VERSION_PROPERTY, formatVersion);

if (icebergTable.properties().containsKey(COMMIT_NUM_RETRIES)) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@
import static java.util.Objects.requireNonNull;
import static org.apache.iceberg.TableProperties.DEFAULT_FILE_FORMAT;
import static org.apache.iceberg.TableProperties.DEFAULT_FILE_FORMAT_DEFAULT;
import static org.apache.iceberg.TableUtil.formatVersion;

public class PropertiesTable
implements SystemTable
Expand Down Expand Up @@ -90,7 +91,7 @@ private static List<Page> buildPages(ConnectorTableMetadata tableMetadata, BaseT
properties.put("provider", "iceberg");
properties.put("current-snapshot-id", currentSnapshotId);
properties.put("location", icebergTable.location());
properties.put("format-version", String.valueOf(icebergTable.operations().current().formatVersion()));
properties.put("format-version", String.valueOf(formatVersion(icebergTable)));
// TODO: Support sort column transforms (https://github.com/trinodb/trino/issues/15088)
SortOrder sortOrder = icebergTable.sortOrder();
if (!sortOrder.isUnsorted() && sortOrder.fields().stream().allMatch(sortField -> sortField.transform().isIdentity())) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -117,6 +117,7 @@
import static org.apache.iceberg.TableProperties.METADATA_DELETE_AFTER_COMMIT_ENABLED;
import static org.apache.iceberg.TableProperties.METADATA_PREVIOUS_VERSIONS_MAX;
import static org.apache.iceberg.TableProperties.SPLIT_SIZE;
import static org.apache.iceberg.TableUtil.formatVersion;
import static org.apache.iceberg.mapping.NameMappingParser.toJson;
import static org.assertj.core.api.Assertions.assertThat;
import static org.junit.jupiter.api.TestInstance.Lifecycle.PER_CLASS;
Expand Down Expand Up @@ -154,11 +155,11 @@ public void testSettingFormatVersion()
{
String tableName = "test_seting_format_version_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 2) AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(2);
assertUpdate("DROP TABLE " + tableName);

assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 1) AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(1);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(1);
assertUpdate("DROP TABLE " + tableName);
}

Expand All @@ -167,7 +168,7 @@ public void testDefaultFormatVersion()
{
String tableName = "test_default_format_version_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(2);
assertUpdate("DROP TABLE " + tableName);
}

Expand Down Expand Up @@ -687,9 +688,9 @@ public void testUpgradeTableToV2FromTrino()
{
String tableName = "test_upgrade_table_to_v2_from_trino_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 1) AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(1);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(1);
assertUpdate("ALTER TABLE " + tableName + " SET PROPERTIES format_version = 2");
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(2);
assertQuery("SELECT * FROM " + tableName, "SELECT * FROM nation");
}

Expand All @@ -698,7 +699,7 @@ public void testDowngradingV2TableToV1Fails()
{
String tableName = "test_downgrading_v2_table_to_v1_fails_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 2) AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(2);
assertThat(query("ALTER TABLE " + tableName + " SET PROPERTIES format_version = 1"))
.failure()
.hasMessage("Failed to set new property values")
Expand All @@ -711,7 +712,7 @@ public void testUpgradingToInvalidVersionFails()
{
String tableName = "test_upgrading_to_invalid_version_fails_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 2) AS SELECT * FROM tpch.tiny.nation", 25);
assertThat(loadTable(tableName).operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(loadTable(tableName))).isEqualTo(2);
assertThat(query("ALTER TABLE " + tableName + " SET PROPERTIES format_version = 42"))
.failure().hasMessage("line 1:79: Unable to set catalog 'iceberg' table property 'format_version' to [42]: format_version must be between 1 and 2");
}
Expand All @@ -722,13 +723,13 @@ public void testUpdatingAllTableProperties()
String tableName = "test_updating_all_table_properties_" + randomNameSuffix();
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 1, format = 'ORC') AS SELECT * FROM tpch.tiny.nation", 25);
BaseTable table = loadTable(tableName);
assertThat(table.operations().current().formatVersion()).isEqualTo(1);
assertThat(formatVersion(table)).isEqualTo(1);
assertThat(table.properties().get(TableProperties.DEFAULT_FILE_FORMAT).equalsIgnoreCase("ORC")).isTrue();
assertThat(table.spec().isUnpartitioned()).isTrue();

assertUpdate("ALTER TABLE " + tableName + " SET PROPERTIES format_version = 2, partitioning = ARRAY['regionkey'], format = 'PARQUET', sorted_by = ARRAY['comment']");
table = loadTable(tableName);
assertThat(table.operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(table)).isEqualTo(2);
assertThat(table.properties().get(TableProperties.DEFAULT_FILE_FORMAT).equalsIgnoreCase("PARQUET")).isTrue();
assertThat(table.spec().isPartitioned()).isTrue();
List<PartitionField> partitionFields = table.spec().fields();
Expand All @@ -749,7 +750,7 @@ public void testUnsettingAllTableProperties()
assertUpdate("CREATE TABLE " + tableName + " WITH (format_version = 1, format = 'PARQUET', partitioning = ARRAY['regionkey'], sorted_by = ARRAY['comment']) " +
"AS SELECT * FROM tpch.tiny.nation", 25);
BaseTable table = loadTable(tableName);
assertThat(table.operations().current().formatVersion()).isEqualTo(1);
assertThat(formatVersion(table)).isEqualTo(1);
assertThat(table.properties().get(TableProperties.DEFAULT_FILE_FORMAT).equalsIgnoreCase("PARQUET")).isTrue();
assertThat(table.spec().isPartitioned()).isTrue();
List<PartitionField> partitionFields = table.spec().fields();
Expand All @@ -759,7 +760,7 @@ public void testUnsettingAllTableProperties()

assertUpdate("ALTER TABLE " + tableName + " SET PROPERTIES format_version = DEFAULT, format = DEFAULT, partitioning = DEFAULT, sorted_by = DEFAULT");
table = loadTable(tableName);
assertThat(table.operations().current().formatVersion()).isEqualTo(2);
assertThat(formatVersion(table)).isEqualTo(2);
assertThat(table.properties().get(TableProperties.DEFAULT_FILE_FORMAT).equalsIgnoreCase("PARQUET")).isTrue();
assertThat(table.spec().isUnpartitioned()).isTrue();
assertThat(table.sortOrder().isUnsorted()).isTrue();
Expand Down