-
Notifications
You must be signed in to change notification settings - Fork 3.6k
Support Nessie Catalog in Iceberg connector #11701
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
Changes from all commits
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -20,5 +20,6 @@ public enum CatalogType | |
| GLUE, | ||
| REST, | ||
| JDBC, | ||
| NESSIE, | ||
| /**/; | ||
| } | ||
| Original file line number | Diff line number | Diff line change |
|---|---|---|
| @@ -0,0 +1,71 @@ | ||
| /* | ||
| * Licensed 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 io.trino.plugin.iceberg.catalog.nessie; | ||
|
|
||
| import io.airlift.configuration.Config; | ||
| import io.airlift.configuration.ConfigDescription; | ||
|
|
||
| import javax.validation.constraints.NotEmpty; | ||
| import javax.validation.constraints.NotNull; | ||
|
|
||
| import java.net.URI; | ||
|
|
||
| public class IcebergNessieCatalogConfig | ||
| { | ||
| private String defaultReferenceName = "main"; | ||
| private String defaultWarehouseDir; | ||
| private URI serverUri; | ||
|
|
||
| @NotNull | ||
| public String getDefaultReferenceName() | ||
| { | ||
| return defaultReferenceName; | ||
| } | ||
|
|
||
| @Config("iceberg.nessie-catalog.ref") | ||
| @ConfigDescription("The default Nessie reference to work on") | ||
| public IcebergNessieCatalogConfig setDefaultReferenceName(String defaultReferenceName) | ||
| { | ||
| this.defaultReferenceName = defaultReferenceName; | ||
| return this; | ||
| } | ||
|
|
||
| @NotNull | ||
| public URI getServerUri() | ||
| { | ||
| return serverUri; | ||
| } | ||
|
|
||
| @Config("iceberg.nessie-catalog.uri") | ||
| @ConfigDescription("The URI to connect to the Nessie server") | ||
| public IcebergNessieCatalogConfig setServerUri(URI serverUri) | ||
| { | ||
| this.serverUri = serverUri; | ||
| return this; | ||
| } | ||
|
|
||
| @NotEmpty | ||
| public String getDefaultWarehouseDir() | ||
| { | ||
| return defaultWarehouseDir; | ||
| } | ||
|
|
||
| @Config("iceberg.nessie-catalog.default-warehouse-dir") | ||
| @ConfigDescription("The default warehouse to use for Nessie") | ||
| public IcebergNessieCatalogConfig setDefaultWarehouseDir(String defaultWarehouseDir) | ||
| { | ||
| this.defaultWarehouseDir = defaultWarehouseDir; | ||
| return this; | ||
| } | ||
| } |
| Original file line number | Diff line number | Diff line change |
|---|---|---|
| @@ -0,0 +1,56 @@ | ||
| /* | ||
| * Licensed 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 io.trino.plugin.iceberg.catalog.nessie; | ||
|
|
||
| import com.google.common.collect.ImmutableMap; | ||
| import com.google.inject.Binder; | ||
| import com.google.inject.Provides; | ||
| import com.google.inject.Scopes; | ||
| import com.google.inject.Singleton; | ||
| import io.airlift.configuration.AbstractConfigurationAwareModule; | ||
| import io.trino.plugin.iceberg.catalog.IcebergTableOperationsProvider; | ||
| import io.trino.plugin.iceberg.catalog.TrinoCatalogFactory; | ||
| import org.apache.iceberg.nessie.NessieIcebergClient; | ||
| import org.projectnessie.client.api.NessieApiV1; | ||
| import org.projectnessie.client.http.HttpClientBuilder; | ||
|
|
||
| import static io.airlift.configuration.ConfigBinder.configBinder; | ||
| import static org.weakref.jmx.guice.ExportBinder.newExporter; | ||
|
|
||
| public class IcebergNessieCatalogModule | ||
| extends AbstractConfigurationAwareModule | ||
| { | ||
| @Override | ||
| protected void setup(Binder binder) | ||
| { | ||
| configBinder(binder).bindConfig(IcebergNessieCatalogConfig.class); | ||
| binder.bind(IcebergTableOperationsProvider.class).to(IcebergNessieTableOperationsProvider.class).in(Scopes.SINGLETON); | ||
| newExporter(binder).export(IcebergTableOperationsProvider.class).withGeneratedName(); | ||
| binder.bind(TrinoCatalogFactory.class).to(TrinoNessieCatalogFactory.class).in(Scopes.SINGLETON); | ||
| newExporter(binder).export(TrinoCatalogFactory.class).withGeneratedName(); | ||
| } | ||
|
|
||
| @Provides | ||
| @Singleton | ||
| public static NessieIcebergClient createNessieIcebergClient(IcebergNessieCatalogConfig icebergNessieCatalogConfig) | ||
| { | ||
| return new NessieIcebergClient( | ||
|
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Is it intentional that the client is built in this initial phase without authentication?
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. currently this is intentional. I didn't want to complicate this PR and add all the different nessie configuration settings (incl authentication). I'm planning to add this as a follow-up |
||
| HttpClientBuilder.builder() | ||
| .withUri(icebergNessieCatalogConfig.getServerUri()) | ||
| .build(NessieApiV1.class), | ||
| icebergNessieCatalogConfig.getDefaultReferenceName(), | ||
| null, | ||
| ImmutableMap.of()); | ||
| } | ||
| } | ||
| Original file line number | Diff line number | Diff line change |
|---|---|---|
| @@ -0,0 +1,133 @@ | ||
| /* | ||
| * Licensed 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 io.trino.plugin.iceberg.catalog.nessie; | ||
|
|
||
| import io.trino.plugin.iceberg.catalog.AbstractIcebergTableOperations; | ||
| import io.trino.spi.TrinoException; | ||
| import io.trino.spi.connector.ConnectorSession; | ||
| import io.trino.spi.connector.SchemaTableName; | ||
| import io.trino.spi.connector.TableNotFoundException; | ||
| import org.apache.iceberg.TableMetadata; | ||
| import org.apache.iceberg.exceptions.CommitFailedException; | ||
| import org.apache.iceberg.io.FileIO; | ||
| import org.apache.iceberg.nessie.NessieIcebergClient; | ||
| import org.projectnessie.error.NessieConflictException; | ||
| import org.projectnessie.error.NessieNotFoundException; | ||
| import org.projectnessie.model.ContentKey; | ||
| import org.projectnessie.model.IcebergTable; | ||
| import org.projectnessie.model.Namespace; | ||
|
|
||
| import java.util.Optional; | ||
|
|
||
| import static com.google.common.base.Verify.verify; | ||
| import static io.trino.plugin.iceberg.IcebergErrorCode.ICEBERG_CATALOG_ERROR; | ||
| import static io.trino.plugin.iceberg.IcebergErrorCode.ICEBERG_COMMIT_ERROR; | ||
| import static io.trino.plugin.iceberg.catalog.nessie.IcebergNessieUtil.toIdentifier; | ||
| import static java.lang.String.format; | ||
| import static java.util.Objects.requireNonNull; | ||
|
|
||
| public class IcebergNessieTableOperations | ||
| extends AbstractIcebergTableOperations | ||
| { | ||
| private final NessieIcebergClient nessieClient; | ||
| private IcebergTable table; | ||
|
|
||
| protected IcebergNessieTableOperations( | ||
| NessieIcebergClient nessieClient, | ||
| FileIO fileIo, | ||
| ConnectorSession session, | ||
| String database, | ||
| String table, | ||
| Optional<String> owner, | ||
| Optional<String> location) | ||
| { | ||
| super(fileIo, session, database, table, owner, location); | ||
| this.nessieClient = requireNonNull(nessieClient, "nessieClient is null"); | ||
| } | ||
|
|
||
| @Override | ||
| public TableMetadata refresh() | ||
| { | ||
| refreshNessieClient(); | ||
| return super.refresh(); | ||
| } | ||
|
|
||
| private void refreshNessieClient() | ||
| { | ||
| try { | ||
| nessieClient.refresh(); | ||
| } | ||
| catch (NessieNotFoundException e) { | ||
| throw new TrinoException(ICEBERG_CATALOG_ERROR, format("Failed to refresh as ref '%s' is no longer valid.", nessieClient.refName()), e); | ||
| } | ||
| } | ||
|
|
||
| @Override | ||
| public TableMetadata refresh(boolean invalidateCaches) | ||
| { | ||
| refreshNessieClient(); | ||
| return super.refresh(invalidateCaches); | ||
| } | ||
|
|
||
| @Override | ||
| protected String getRefreshedLocation(boolean invalidateCaches) | ||
| { | ||
| table = nessieClient.table(toIdentifier(new SchemaTableName(database, tableName))); | ||
|
|
||
| if (table == null) { | ||
| throw new TableNotFoundException(getSchemaTableName()); | ||
| } | ||
|
|
||
| return table.getMetadataLocation(); | ||
| } | ||
|
|
||
| @Override | ||
| protected void commitNewTable(TableMetadata metadata) | ||
| { | ||
| verify(version.isEmpty(), "commitNewTable called on a table which already exists"); | ||
| try { | ||
| nessieClient.commitTable(null, metadata, writeNewMetadata(metadata, 0), table, toKey(new SchemaTableName(database, this.tableName))); | ||
| } | ||
| catch (NessieNotFoundException e) { | ||
| throw new TrinoException(ICEBERG_COMMIT_ERROR, format("Cannot commit: ref '%s' no longer exists", nessieClient.refName()), e); | ||
| } | ||
| catch (NessieConflictException e) { | ||
| // CommitFailedException is handled as a special case in the Iceberg library. This commit will automatically retry | ||
| throw new CommitFailedException(e, "Cannot commit: ref hash is out of date. Update the ref '%s' and try again", nessieClient.refName()); | ||
| } | ||
| shouldRefresh = true; | ||
| } | ||
|
|
||
| @Override | ||
| protected void commitToExistingTable(TableMetadata base, TableMetadata metadata) | ||
| { | ||
| verify(version.orElseThrow() >= 0, "commitToExistingTable called on a new table"); | ||
| try { | ||
| nessieClient.commitTable(base, metadata, writeNewMetadata(metadata, version.getAsInt() + 1), table, toKey(new SchemaTableName(database, this.tableName))); | ||
| } | ||
| catch (NessieNotFoundException e) { | ||
| throw new TrinoException(ICEBERG_COMMIT_ERROR, format("Cannot commit: ref '%s' no longer exists", nessieClient.refName()), e); | ||
| } | ||
| catch (NessieConflictException e) { | ||
| // CommitFailedException is handled as a special case in the Iceberg library. This commit will automatically retry | ||
| throw new CommitFailedException(e, "Cannot commit: ref hash is out of date. Update the ref '%s' and try again", nessieClient.refName()); | ||
| } | ||
| shouldRefresh = true; | ||
| } | ||
|
|
||
| private static ContentKey toKey(SchemaTableName tableName) | ||
| { | ||
| return ContentKey.of(Namespace.parse(tableName.getSchemaName()), tableName.getTableName()); | ||
| } | ||
| } |
Uh oh!
There was an error while loading. Please reload this page.