dimas-b commented on code in PR #8909: URL: https://github.com/apache/iceberg/pull/8909#discussion_r1422564943
########## nessie/src/main/java/org/apache/iceberg/nessie/NessieTableOperations.java: ########## @@ -133,69 +114,30 @@ protected void doCommit(TableMetadata base, TableMetadata metadata) { boolean newTable = base == null; String newMetadataLocation = writeNewMetadataIfRequired(newTable, metadata); - String refName = client.refName(); boolean failure = false; try { String contentId = table == null ? null : table.getId(); client.commitTable(base, metadata, newMetadataLocation, contentId, key); - } catch (NessieConflictException ex) { - failure = true; - if (ex instanceof NessieReferenceConflictException) { - // Throws a specialized exception, if possible - maybeThrowSpecializedException((NessieReferenceConflictException) ex); + } catch (NessieConflictException | NessieNotFoundException | HttpClientException ex) { + if (ex instanceof NessieConflictException || ex instanceof NessieNotFoundException) { + failure = true; } - throw new CommitFailedException( - ex, - "Cannot commit: Reference hash is out of date. " - + "Update the reference '%s' and try again", - refName); - } catch (HttpClientException ex) { - // Intentionally catch all nessie-client-exceptions here and not just the "timeout" variant - // to catch all kinds of network errors (e.g. connection reset). Network code implementation - // details and all kinds of network devices can induce unexpected behavior. So better be - // safe than sorry. - throw new CommitStateUnknownException(ex); - } catch (NessieNotFoundException ex) { + NessieUtil.handleExceptionsForCommits(ex, client.refName(), Content.Type.ICEBERG_TABLE) + .ifPresent( + exception -> { + throw exception; Review Comment: If I follow the logic correctly, `handleExceptionsForCommits` will always return something for the exception types caught in this case, so `ifPresent()` is really confusing. Could the code be refactored to avoid unnecessary conditional execution? ########## nessie/src/main/java/org/apache/iceberg/nessie/NessieViewOperations.java: ########## @@ -0,0 +1,129 @@ +/* + * 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.iceberg.nessie; + +import org.apache.iceberg.exceptions.NoSuchViewException; +import org.apache.iceberg.io.FileIO; +import org.apache.iceberg.view.BaseViewOperations; +import org.apache.iceberg.view.ViewMetadata; +import org.apache.iceberg.view.ViewMetadataParser; +import org.projectnessie.client.http.HttpClientException; +import org.projectnessie.error.NessieBadRequestException; +import org.projectnessie.error.NessieConflictException; +import org.projectnessie.error.NessieContentNotFoundException; +import org.projectnessie.error.NessieNotFoundException; +import org.projectnessie.model.Content; +import org.projectnessie.model.ContentKey; +import org.projectnessie.model.IcebergView; +import org.projectnessie.model.Reference; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class NessieViewOperations extends BaseViewOperations { + + private static final Logger LOG = LoggerFactory.getLogger(NessieViewOperations.class); + + private final NessieIcebergClient client; + private final ContentKey key; + private final FileIO fileIO; + private IcebergView icebergView; + + NessieViewOperations(ContentKey key, NessieIcebergClient client, FileIO fileIO) { + this.key = key; + this.client = client; + this.fileIO = fileIO; + } + + @Override + public void doRefresh() { + try { + client.refresh(); + } catch (NessieNotFoundException e) { + throw new RuntimeException( + String.format( + "Failed to refresh as ref '%s' is no longer valid.", client.getRef().getName()), + e); + } + String metadataLocation = null; + Reference reference = client.getRef().getReference(); + try { + Content content = client.getApi().getContent().key(key).reference(reference).get().get(key); + LOG.debug("Content '{}' at '{}': {}", key, reference, content); + if (content == null) { + if (currentMetadataLocation() != null) { + throw new NoSuchViewException("View does not exist: %s in %s", key, reference); + } + } else { + this.icebergView = + content + .unwrap(IcebergView.class) + .orElseThrow(() -> new NessieContentNotFoundException(key, reference.getName())); + metadataLocation = icebergView.getMetadataLocation(); + } + } catch (NessieNotFoundException ex) { + if (currentMetadataLocation() != null) { + throw new NoSuchViewException("View does not exist: %s in %s", key, reference); + } + } + refreshFromMetadataLocation( + metadataLocation, + null, + 2, + location -> + NessieUtil.loadViewMetadata( + ViewMetadataParser.read(io().newInputFile(location)), location, reference)); + } + + @Override + public void doCommit(ViewMetadata base, ViewMetadata metadata) { + String newMetadataLocation = writeNewMetadataIfRequired(metadata); + + boolean failure = false; + try { + String contentId = icebergView == null ? null : icebergView.getId(); + client.commitView(base, metadata, newMetadataLocation, contentId, key); + } catch (NessieConflictException | NessieNotFoundException | HttpClientException ex) { + if (ex instanceof NessieConflictException || ex instanceof NessieNotFoundException) { + failure = true; + } + NessieUtil.handleExceptionsForCommits(ex, client.refName(), Content.Type.ICEBERG_VIEW) + .ifPresent( + exception -> { + throw exception; + }); + } catch (NessieBadRequestException ex) { + failure = true; + throw NessieUtil.handleBadRequestForCommit(client, key, Content.Type.ICEBERG_VIEW).orElse(ex); + } finally { + if (failure) { + io().deleteFile(newMetadataLocation); Review Comment: This code appears to be shared with `NessieTableOperations`. Could we refactor it to have the exception/failure/cleanup logic in one place? ########## nessie/src/test/java/org/apache/iceberg/nessie/BaseTestIceberg.java: ########## @@ -267,4 +305,23 @@ static DataFile makeDataFile(Table icebergTable, String fileLocation) { .withFileSizeInBytes(Files.localInput(fileLocation).getLength()) .build(); } + + protected static List<String> metadataVersionFiles(String tablePath) { + return filterByExtension(tablePath, getFileExtension(TableMetadataParser.Codec.NONE)); + } + + protected static List<String> filterByExtension(String tablePath, String extension) { + return metadataFiles(tablePath).stream() + .filter(f -> f.endsWith(extension)) + .collect(Collectors.toList()); + } + + @SuppressWarnings( + "RegexpSinglelineJava") // respecting this rule requires a lot more lines of code Review Comment: nit: I _think_ if you moved the comment to a separate like, the annotation then would format as one line too. ########## nessie/src/test/java/org/apache/iceberg/nessie/TestNessieViewCatalog.java: ########## @@ -0,0 +1,220 @@ +/* + * 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.iceberg.nessie; + +import static org.assertj.core.api.Assertions.assertThat; + +import java.io.IOException; +import java.net.URI; +import java.nio.file.Path; +import org.apache.hadoop.conf.Configuration; +import org.apache.iceberg.CatalogProperties; +import org.apache.iceberg.catalog.Catalog; +import org.apache.iceberg.catalog.TableIdentifier; +import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap; +import org.apache.iceberg.view.BaseView; +import org.apache.iceberg.view.View; +import org.apache.iceberg.view.ViewCatalogTests; +import org.apache.iceberg.view.ViewMetadata; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.BeforeEach; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.junit.jupiter.api.extension.RegisterExtension; +import org.junit.jupiter.api.io.TempDir; +import org.projectnessie.client.api.NessieApiV1; +import org.projectnessie.client.ext.NessieApiVersion; +import org.projectnessie.client.ext.NessieApiVersions; +import org.projectnessie.client.ext.NessieClientFactory; +import org.projectnessie.client.ext.NessieClientUri; +import org.projectnessie.error.NessieConflictException; +import org.projectnessie.error.NessieNotFoundException; +import org.projectnessie.jaxrs.ext.NessieJaxRsExtension; +import org.projectnessie.model.Branch; +import org.projectnessie.model.Reference; +import org.projectnessie.model.Tag; +import org.projectnessie.versioned.storage.common.persist.Persist; +import org.projectnessie.versioned.storage.inmemory.InmemoryBackendTestFactory; +import org.projectnessie.versioned.storage.testextension.NessieBackend; +import org.projectnessie.versioned.storage.testextension.NessiePersist; +import org.projectnessie.versioned.storage.testextension.PersistExtension; + +@ExtendWith(PersistExtension.class) +@NessieBackend(InmemoryBackendTestFactory.class) +@NessieApiVersions // test all versions +public class TestNessieViewCatalog extends ViewCatalogTests<NessieCatalog> { + + @NessiePersist static Persist persist; + + @RegisterExtension + static NessieJaxRsExtension server = NessieJaxRsExtension.jaxRsExtension(() -> persist); + + @TempDir private Path temp; + + private NessieCatalog catalog; + private NessieApiV1 api; + private NessieApiVersion apiVersion; + private Configuration hadoopConfig; + private String initialHashOfDefaultBranch; + private String uri; + + @BeforeEach + public void setUp(NessieClientFactory clientFactory, @NessieClientUri URI nessieUri) + throws NessieNotFoundException { + api = clientFactory.make(); + apiVersion = clientFactory.apiVersion(); + initialHashOfDefaultBranch = api.getDefaultBranch().getHash(); + uri = nessieUri.toASCIIString(); + hadoopConfig = new Configuration(); + catalog = initNessieCatalog("main"); + } + + @AfterEach + public void afterEach() throws IOException { + resetData(); + try { + if (catalog != null) { + catalog.close(); + } + api.close(); + } finally { + catalog = null; + api = null; + hadoopConfig = null; + } + } + + private void resetData() throws NessieConflictException, NessieNotFoundException { + Branch defaultBranch = api.getDefaultBranch(); + for (Reference r : api.getAllReferences().get().getReferences()) { Review Comment: tangential: This code is repeated in many downstream tests... Perhaps we could make a helper utility in Nessie OSS :thinking: ########## nessie/src/main/java/org/apache/iceberg/nessie/NessieIcebergClient.java: ########## @@ -448,33 +480,85 @@ public void renameTable(TableIdentifier from, TableIdentifier to) { // behavior. So better be safe than sorry. } + private static void validateToContentForRename( + TableIdentifier from, TableIdentifier to, IcebergContent existingToContent) { + if (existingToContent != null) { + if (existingToContent.getType() == Content.Type.ICEBERG_VIEW) { + throw new AlreadyExistsException("Cannot rename %s to %s. View already exists", from, to); + } else if (existingToContent.getType() == Content.Type.ICEBERG_TABLE) { + throw new AlreadyExistsException("Cannot rename %s to %s. Table already exists", from, to); + } else { + throw new AlreadyExistsException( + "Cannot rename %s to %s. Another content of type %s with same name already exists", + from, to, existingToContent.getType()); + } + } + } + + private static void validateFromContentForRename( + TableIdentifier from, Content.Type type, IcebergContent existingFromContent) { + if (existingFromContent == null) { + if (type == Content.Type.ICEBERG_VIEW) { + throw new NoSuchViewException("View does not exist: %s", from); + } else if (type == Content.Type.ICEBERG_TABLE) { + throw new NoSuchTableException("Table does not exist: %s", from); + } else { + throw new RuntimeException("Cannot perform rename for content type: " + type); + } + } else if (existingFromContent.getType() != type) { + throw new RuntimeException( + String.format("content type of from identifier %s should be of %s", from, type)); + } + } + public boolean dropTable(TableIdentifier identifier, boolean purge) { + return dropContent(identifier, purge, Content.Type.ICEBERG_TABLE); + } + + public boolean dropView(TableIdentifier identifier, boolean purge) { + return dropContent(identifier, purge, Content.Type.ICEBERG_VIEW); + } + + private boolean dropContent(TableIdentifier identifier, boolean purge, Content.Type type) { getRef().checkMutable(); - IcebergTable existingTable = table(identifier); - if (existingTable == null) { + IcebergContent existingContent = fetchContent(identifier); + if (existingContent == null) { return false; } + if (existingContent.getType() != type) { + throw new RuntimeException( + String.format( + "Cannot drop %s: not matching with the type `%s`", + identifier, NessieUtil.contentTypeString(type))); + } + + String contentType = NessieUtil.contentTypeString(type).toLowerCase(Locale.ENGLISH); + if (purge) { - LOG.info("Purging data for table {} was set to true but is ignored", identifier.toString()); + LOG.info( + "Purging data for {} {} was set to true but is ignored", + contentType, + identifier.toString()); } - // We try to drop the table. Simple retry after ref update. + // We try to drop the content. Simple retry after ref update. try { commitRetry( String.format("Iceberg delete table %s", identifier), Review Comment: This may be a view now, not just "table" (cf. line 554). ########## nessie/src/main/java/org/apache/iceberg/nessie/NessieTableOperations.java: ########## @@ -133,69 +114,30 @@ protected void doCommit(TableMetadata base, TableMetadata metadata) { boolean newTable = base == null; String newMetadataLocation = writeNewMetadataIfRequired(newTable, metadata); - String refName = client.refName(); boolean failure = false; try { String contentId = table == null ? null : table.getId(); client.commitTable(base, metadata, newMetadataLocation, contentId, key); - } catch (NessieConflictException ex) { - failure = true; - if (ex instanceof NessieReferenceConflictException) { - // Throws a specialized exception, if possible - maybeThrowSpecializedException((NessieReferenceConflictException) ex); + } catch (NessieConflictException | NessieNotFoundException | HttpClientException ex) { + if (ex instanceof NessieConflictException || ex instanceof NessieNotFoundException) { + failure = true; Review Comment: I think it might be preferable to delegate determining the `failure` flag to `NessieUtil.handleExceptionsForCommits`. This logic should be the same between tables and views. Perhaps `handleExceptionsForCommits` could return an object with both the flag and the exception to be re-thrown. WDYT? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: issues-unsubscr...@iceberg.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org --------------------------------------------------------------------- To unsubscribe, e-mail: issues-unsubscr...@iceberg.apache.org For additional commands, e-mail: issues-h...@iceberg.apache.org