Skip to content

Upgrade Polaris to Iceberg 1.9.0 #1309

New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Merged
merged 2 commits into from
Apr 28, 2025
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
2 changes: 1 addition & 1 deletion gradle/libs.versions.toml
Original file line number Diff line number Diff line change
Expand Up @@ -19,7 +19,7 @@

[versions]
hadoop = "3.4.1"
iceberg = "1.8.1" # Ensure to update the iceberg version in regtests to keep regtests up-to-date
iceberg = "1.9.0" # Ensure to update the iceberg version in regtests to keep regtests up-to-date
quarkus = "3.21.2"
immutables = "2.10.1"
picocli = "4.7.7"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,8 +20,6 @@

import com.google.common.collect.ImmutableMap;
import java.util.Map;
import org.apache.iceberg.catalog.SessionCatalog;
import org.apache.iceberg.rest.HTTPClient;
import org.apache.iceberg.rest.RESTCatalog;
import org.apache.iceberg.rest.auth.OAuth2Properties;
import org.apache.polaris.core.admin.model.PrincipalWithCredentials;
Expand All @@ -36,14 +34,7 @@ public static RESTCatalog restCatalog(
String catalog,
Map<String, String> extraProperties) {
String authToken = client.obtainToken(credentials);
SessionCatalog.SessionContext context = SessionCatalog.SessionContext.createEmpty();
RESTCatalog restCatalog =
new RESTCatalog(
context,
(config) ->
HTTPClient.builder(config)
.uri(config.get(org.apache.iceberg.CatalogProperties.URI))
.build());
RESTCatalog restCatalog = new RESTCatalog();

ImmutableMap.Builder<String, String> propertiesBuilder =
ImmutableMap.<String, String>builder()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -371,8 +371,8 @@ public void testIcebergCreateTablesInExternalCatalog() throws IOException {
TableIdentifier.of(ns, "the_table"),
new Schema(
List.of(
Types.NestedField.of(
1, false, "theField", Types.StringType.get()))))
Types.NestedField.required(
1, "theField", Types.StringType.get()))))
.withLocation("file:///tmp/tables")
.withSortOrder(SortOrder.unsorted())
.withPartitionSpec(PartitionSpec.unpartitioned())
Expand All @@ -399,8 +399,8 @@ public void testIcebergCreateTablesWithWritePathBlocked() throws IOException {
TableIdentifier.of(ns, "the_table"),
new Schema(
List.of(
Types.NestedField.of(
1, false, "theField", Types.StringType.get()))))
Types.NestedField.required(
1, "theField", Types.StringType.get()))))
.withSortOrder(SortOrder.unsorted())
.withPartitionSpec(PartitionSpec.unpartitioned())
.withProperties(Map.of("write.data.path", "s3://my-bucket/path/to/data"))
Expand All @@ -416,8 +416,8 @@ public void testIcebergCreateTablesWithWritePathBlocked() throws IOException {
TableIdentifier.of(ns, "the_table"),
new Schema(
List.of(
Types.NestedField.of(
1, false, "theField", Types.StringType.get()))))
Types.NestedField.required(
1, "theField", Types.StringType.get()))))
.withSortOrder(SortOrder.unsorted())
.withPartitionSpec(PartitionSpec.unpartitioned())
.withProperties(Map.of("write.metadata.path", "s3://my-bucket/path/to/data"))
Expand Down Expand Up @@ -454,7 +454,7 @@ public void testIcebergRegisterTableInExternalCatalog() throws IOException {
.assignUUID()
.addPartitionSpec(PartitionSpec.unpartitioned())
.addSortOrder(SortOrder.unsorted())
.addSchema(new Schema(Types.NestedField.of(1, false, "col1", Types.StringType.get())))
.addSchema(new Schema(Types.NestedField.required(1, "col1", Types.StringType.get())))
.build();
TableMetadataParser.write(tableMetadata, fileIo.newOutputFile(metadataLocation));

Expand Down Expand Up @@ -490,7 +490,7 @@ public void testIcebergUpdateTableInExternalCatalog() throws IOException {
String location = baseLocation.resolve("testIcebergUpdateTableInExternalCatalog").toString();
String metadataLocation = location + "/metadata/000001-494949494949494949.metadata.json";

Types.NestedField col1 = Types.NestedField.of(1, false, "col1", Types.StringType.get());
Types.NestedField col1 = Types.NestedField.required(1, "col1", Types.StringType.get());
TableMetadata tableMetadata =
TableMetadata.buildFromEmpty()
.setLocation(location)
Expand Down Expand Up @@ -545,7 +545,7 @@ public void testIcebergDropTableInExternalCatalog() throws IOException {
.assignUUID()
.addPartitionSpec(PartitionSpec.unpartitioned())
.addSortOrder(SortOrder.unsorted())
.addSchema(new Schema(Types.NestedField.of(1, false, "col1", Types.StringType.get())))
.addSchema(new Schema(Types.NestedField.required(1, "col1", Types.StringType.get())))
.build();
TableMetadataParser.write(tableMetadata, fileIo.newOutputFile(metadataLocation));

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -148,6 +148,19 @@ public class PolarisRestCatalogIntegrationTest extends CatalogTests<RESTCatalog>
private final String catalogBaseLocation =
s3BucketBase + "/" + System.getenv("USER") + "/path/to/data";

private static final Map<String, String> DEFAULT_REST_CATALOG_CONFIG =
Map.of(
org.apache.iceberg.CatalogProperties.TABLE_DEFAULT_PREFIX + "default-key1",
"catalog-default-key1",
org.apache.iceberg.CatalogProperties.TABLE_DEFAULT_PREFIX + "default-key2",
"catalog-default-key2",
org.apache.iceberg.CatalogProperties.TABLE_DEFAULT_PREFIX + "override-key3",
"catalog-default-key3",
org.apache.iceberg.CatalogProperties.TABLE_OVERRIDE_PREFIX + "override-key3",
"catalog-override-key3",
org.apache.iceberg.CatalogProperties.TABLE_OVERRIDE_PREFIX + "override-key4",
"catalog-override-key4");

private static final String[] DEFAULT_CATALOG_PROPERTIES = {
"allow.unstructured.table.location", "true",
"allow.external.table.location", "true"
Expand Down Expand Up @@ -235,7 +248,7 @@ public void before(TestInfo testInfo) {

managementApi.createCatalog(principalRoleName, catalog);

restCatalogConfig =
Map<String, String> dynamicConfig =
testInfo
.getTestMethod()
.map(m -> m.getAnnotation(RestCatalogConfig.class))
Expand All @@ -254,6 +267,12 @@ public void before(TestInfo testInfo) {
})
.orElse(ImmutableMap.of());

restCatalogConfig =
ImmutableMap.<String, String>builder()
.putAll(DEFAULT_REST_CATALOG_CONFIG)
.putAll(dynamicConfig)
.build();

restCatalog = initCatalog(currentCatalogName, ImmutableMap.of());
}

Expand Down Expand Up @@ -581,7 +600,7 @@ public void testLoadTableWithAccessDelegationForExternalCatalogWithConfigDisable
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
externalCatalogBase + "/ns1/my_table",
Map.of());
Expand Down Expand Up @@ -616,7 +635,7 @@ public void testLoadTableWithoutAccessDelegationForExternalCatalogWithConfigDisa
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
externalCatalogBase + "/ns1/my_table",
Map.of());
Expand Down Expand Up @@ -650,7 +669,7 @@ public void testLoadTableWithAccessDelegationForExternalCatalogWithConfigEnabled
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
externalCatalogBase + "/ns1/my_table",
Map.of());
Expand Down Expand Up @@ -681,7 +700,7 @@ public void testLoadTableTwiceWithETag() {
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
"file:///tmp/ns1/my_table",
Map.of());
Expand Down Expand Up @@ -728,7 +747,7 @@ public void testRegisterAndLoadTableWithReturnedETag() {
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
"file:///tmp/ns1/my_table",
Map.of());
Expand Down Expand Up @@ -774,7 +793,7 @@ public void testCreateAndLoadTableWithReturnedEtag() {
restCatalog.createNamespace(ns1);
TableMetadata tableMetadata =
TableMetadata.newTableMetadata(
new Schema(List.of(Types.NestedField.of(1, false, "col1", new Types.StringType()))),
new Schema(List.of(Types.NestedField.required(1, "col1", new Types.StringType()))),
PartitionSpec.unpartitioned(),
"file:///tmp/ns1/my_table",
Map.of());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -62,6 +62,16 @@ public abstract class PolarisRestCatalogViewIntegrationBase extends ViewCatalogT
Assumptions.setPreferredAssumptionException(PreferredAssumptionException.JUNIT5);
}

public static Map<String, String> DEFAULT_REST_CATALOG_CONFIG =
Map.of(
org.apache.iceberg.CatalogProperties.VIEW_DEFAULT_PREFIX + "key1", "catalog-default-key1",
org.apache.iceberg.CatalogProperties.VIEW_DEFAULT_PREFIX + "key2", "catalog-default-key2",
org.apache.iceberg.CatalogProperties.VIEW_DEFAULT_PREFIX + "key3", "catalog-default-key3",
org.apache.iceberg.CatalogProperties.VIEW_OVERRIDE_PREFIX + "key3",
"catalog-override-key3",
org.apache.iceberg.CatalogProperties.VIEW_OVERRIDE_PREFIX + "key4",
"catalog-override-key4");

private static ClientCredentials adminCredentials;
private static PolarisApiEndpoints endpoints;
private static PolarisClient client;
Expand Down Expand Up @@ -120,15 +130,7 @@ public void before(TestInfo testInfo) {

restCatalog =
IcebergHelper.restCatalog(
client,
endpoints,
principalCredentials,
catalogName,
Map.of(
org.apache.iceberg.CatalogProperties.VIEW_DEFAULT_PREFIX + "key1",
"catalog-default-key1",
org.apache.iceberg.CatalogProperties.VIEW_DEFAULT_PREFIX + "key2",
"catalog-default-key2"));
client, endpoints, principalCredentials, catalogName, DEFAULT_REST_CATALOG_CONFIG);
}

@AfterEach
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,7 @@
import java.util.Arrays;
import java.util.Map;
import org.apache.iceberg.exceptions.BadRequestException;
import org.apache.iceberg.exceptions.NamespaceNotEmptyException;
import org.apache.iceberg.spark.SupportsReplaceView;
import org.apache.spark.sql.catalyst.analysis.NoSuchNamespaceException;
import org.apache.spark.sql.catalyst.analysis.NoSuchViewException;
Expand Down Expand Up @@ -107,7 +108,10 @@ void testNamespaceOperations() throws Exception {

// directly drop lv1ns[0] should fail
assertThatThrownBy(() -> namespaceCatalog.dropNamespace(lv1ns[0], true))
.isInstanceOf(BadRequestException.class);
.isInstanceOfAny(
BadRequestException.class, // Iceberg < 1.9.0
NamespaceNotEmptyException.class // Iceberg >= 1.9.0
);
for (String[] namespace : lv2ns1) {
namespaceCatalog.dropNamespace(namespace, true);
}
Expand Down Expand Up @@ -249,7 +253,10 @@ void testListViews() throws Exception {

// drop namespace fails since there are views under it
assertThatThrownBy(() -> namespaceCatalog.dropNamespace(l2ns, true))
.isInstanceOf(BadRequestException.class);
.isInstanceOfAny(
BadRequestException.class, // Iceberg < 1.9.0
NamespaceNotEmptyException.class // Iceberg >= 1.9.0
);
// drop the views
for (String name : nsl2ViewNames) {
viewCatalog.dropView(Identifier.of(l2ns, name));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -180,6 +180,19 @@ public Map<String, String> getConfigOverrides() {
public static final String SECRET_ACCESS_KEY = "secret_access_key";
public static final String SESSION_TOKEN = "session_token";

public static Map<String, String> TABLE_PREFIXES =
Map.of(
CatalogProperties.TABLE_DEFAULT_PREFIX + "default-key1",
"catalog-default-key1",
CatalogProperties.TABLE_DEFAULT_PREFIX + "default-key2",
"catalog-default-key2",
CatalogProperties.TABLE_DEFAULT_PREFIX + "override-key3",
"catalog-default-key3",
CatalogProperties.TABLE_OVERRIDE_PREFIX + "override-key3",
"catalog-override-key3",
CatalogProperties.TABLE_OVERRIDE_PREFIX + "override-key4",
"catalog-override-key4");

@Inject MetaStoreManagerFactory managerFactory;
@Inject PolarisConfigurationStore configurationStore;
@Inject PolarisStorageIntegrationProvider storageIntegrationProvider;
Expand Down Expand Up @@ -344,6 +357,7 @@ protected IcebergCatalog initCatalog(
ImmutableMap.Builder<String, String> propertiesBuilder =
ImmutableMap.<String, String>builder()
.put(CatalogProperties.FILE_IO_IMPL, "org.apache.iceberg.inmemory.InMemoryFileIO")
.putAll(TABLE_PREFIXES)
.putAll(additionalProperties);
icebergCatalog.initialize(CATALOG_NAME, propertiesBuilder.buildKeepingLast());
return icebergCatalog;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -104,6 +104,14 @@ public Map<String, String> getConfigOverrides() {

public static final String CATALOG_NAME = "polaris-catalog";

public static Map<String, String> VIEW_PREFIXES =
Map.of(
CatalogProperties.VIEW_DEFAULT_PREFIX + "key1", "catalog-default-key1",
CatalogProperties.VIEW_DEFAULT_PREFIX + "key2", "catalog-default-key2",
CatalogProperties.VIEW_DEFAULT_PREFIX + "key3", "catalog-default-key3",
CatalogProperties.VIEW_OVERRIDE_PREFIX + "key3", "catalog-override-key3",
CatalogProperties.VIEW_OVERRIDE_PREFIX + "key4", "catalog-override-key4");

@Inject MetaStoreManagerFactory managerFactory;
@Inject UserSecretsManagerFactory userSecretsManagerFactory;
@Inject PolarisConfigurationStore configurationStore;
Expand Down Expand Up @@ -211,15 +219,12 @@ public void before(TestInfo testInfo) {
securityContext,
Mockito.mock(),
fileIOFactory);
this.catalog.initialize(
CATALOG_NAME,
ImmutableMap.of(
CatalogProperties.FILE_IO_IMPL,
"org.apache.iceberg.inmemory.InMemoryFileIO",
CatalogProperties.VIEW_DEFAULT_PREFIX + "key1",
"catalog-default-key1",
CatalogProperties.VIEW_DEFAULT_PREFIX + "key2",
"catalog-default-key2"));
Map<String, String> properties =
ImmutableMap.<String, String>builder()
.put(CatalogProperties.FILE_IO_IMPL, "org.apache.iceberg.inmemory.InMemoryFileIO")
.putAll(VIEW_PREFIXES)
.build();
this.catalog.initialize(CATALOG_NAME, properties);
}

@AfterEach
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -100,7 +100,7 @@ static TableMetadata writeTableMetadata(
tmBuilder
.setLocation("path/to/table")
.addSchema(
new Schema(List.of(Types.NestedField.of(1, false, "field1", Types.StringType.get()))))
new Schema(List.of(Types.NestedField.required(1, "field1", Types.StringType.get()))))
.addSortOrder(SortOrder.unsorted())
.assignUUID(UUID.randomUUID().toString())
.addPartitionSpec(PartitionSpec.unpartitioned());
Expand Down
2 changes: 1 addition & 1 deletion regtests/setup.sh
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,7 @@ if [ -z "${SPARK_HOME}" ]; then
fi
SPARK_CONF="${SPARK_HOME}/conf/spark-defaults.conf"
DERBY_HOME="/tmp/derby"
ICEBERG_VERSION="1.8.1"
ICEBERG_VERSION="1.9.0"
export PYTHONPATH="${SPARK_HOME}/python/:${SPARK_HOME}/python/lib/py4j-0.10.9.7-src.zip:$PYTHONPATH"

# Ensure binaries are downloaded locally
Expand Down
2 changes: 1 addition & 1 deletion regtests/t_pyspark/src/iceberg_spark.py
Original file line number Diff line number Diff line change
Expand Up @@ -74,7 +74,7 @@ def __enter__(self):
"""Initial method for Iceberg Spark session. Creates a Spark session with specified configs.
"""
packages = [
"org.apache.iceberg:iceberg-spark-runtime-3.5_2.12:1.8.1",
"org.apache.iceberg:iceberg-spark-runtime-3.5_2.12:1.9.0",
"org.apache.hadoop:hadoop-aws:3.4.0",
"software.amazon.awssdk:bundle:2.23.19",
"software.amazon.awssdk:url-connection-client:2.23.19",
Expand Down