prashantwason commented on code in PR #8837: URL: https://github.com/apache/hudi/pull/8837#discussion_r1245896736
########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -966,8 +966,8 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { // String syncCommitTime = HoodieTableMetadataUtil.createIndexInitTimestamp(HoodieActiveTimeline.createNewInstantTime()); // TODO: Using METADATA_INDEXER_TIME_SUFFIX for now, but this should have its own suffix. To be fixed after HUDI-6200 String syncCommitTime = HoodieActiveTimeline.createNewInstantTime() + METADATA_INDEXER_TIME_SUFFIX; Review Comment: Should use a unique suffix for this. ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -984,52 +984,46 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { */ @Override public void update(HoodieRollbackMetadata rollbackMetadata, String instantTime) { - // The commit which is being rolled back on the dataset - final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); - // Find the deltacommits since the last compaction - Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = - CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); - if (!deltaCommitsInfo.isPresent()) { - LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); - return; - } - - // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) - HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); - HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); - - // The deltacommit that will be rolled back - HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); - - // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions - // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. - if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { - final String compactionInstantTime = compactionInstant.getTimestamp(); - if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { - throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " - + "There are %d deltacommits after this compaction: %s", commitInstantTime, compactionInstantTime, - deltacommitsSinceCompaction.countInstants(), deltacommitsSinceCompaction.getInstants())); + if (initialized && metadata != null) { Review Comment: Why is this check required? We should not be calling these updateXX APIs if the HoodieBackedTableMetadata is not initialized already. ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -984,52 +984,46 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { */ @Override public void update(HoodieRollbackMetadata rollbackMetadata, String instantTime) { - // The commit which is being rolled back on the dataset - final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); - // Find the deltacommits since the last compaction - Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = - CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); - if (!deltaCommitsInfo.isPresent()) { - LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); - return; - } - - // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) - HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); - HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); - - // The deltacommit that will be rolled back - HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); - - // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions - // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. - if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { - final String compactionInstantTime = compactionInstant.getTimestamp(); - if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { - throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " - + "There are %d deltacommits after this compaction: %s", commitInstantTime, compactionInstantTime, - deltacommitsSinceCompaction.countInstants(), deltacommitsSinceCompaction.getInstants())); + if (initialized && metadata != null) { + // The commit which is being rolled back on the dataset + final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); + // Find the deltacommits since the last compaction + Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = + CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); + if (!deltaCommitsInfo.isPresent() || deltaCommitsInfo.get().getKey().empty()) { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); + return; } - } - if (deltacommitsSinceCompaction.containsInstant(deltaCommitInstant)) { - LOG.info("Rolling back MDT deltacommit " + commitInstantTime); - if (!getWriteClient().rollback(commitInstantTime, instantTime)) { - throw new HoodieMetadataException("Failed to rollback deltacommit at " + commitInstantTime); + // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) + HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); + HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); + + // The deltacommit that will be rolled back + HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); + + // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions + // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. + if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { + final String compactionInstantTime = compactionInstant.getTimestamp(); + if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { + throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " Review Comment: oldest compaction ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -914,26 +982,49 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { */ @Override public void update(HoodieRollbackMetadata rollbackMetadata, String instantTime) { - if (initialized && metadata != null) { - // Is this rollback of an instant that has been synced to the metadata table? - String rollbackInstant = rollbackMetadata.getCommitsRollback().get(0); - boolean wasSynced = metadataMetaClient.getActiveTimeline().containsInstant(new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, rollbackInstant)); - if (!wasSynced) { - // A compaction may have taken place on metadata table which would have included this instant being rolled back. - // Revisit this logic to relax the compaction fencing : https://issues.apache.org/jira/browse/HUDI-2458 - Option<String> latestCompaction = metadata.getLatestCompactionTime(); - if (latestCompaction.isPresent()) { - wasSynced = HoodieTimeline.compareTimestamps(rollbackInstant, HoodieTimeline.LESSER_THAN_OR_EQUALS, latestCompaction.get()); - } + // The commit which is being rolled back on the dataset + final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); + // Find the deltacommits since the last compaction + Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = + CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); + if (!deltaCommitsInfo.isPresent()) { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); + return; + } + + // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) + HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); + HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); + + // The deltacommit that will be rolled back + HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); + + // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions + // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. + if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { + final String compactionInstantTime = compactionInstant.getTimestamp(); + if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { + throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " + + "There are %d deltacommits after this compaction: %s", commitInstantTime, compactionInstantTime, + deltacommitsSinceCompaction.countInstants(), deltacommitsSinceCompaction.getInstants())); } + } - Map<MetadataPartitionType, HoodieData<HoodieRecord>> records = - HoodieTableMetadataUtil.convertMetadataToRecords(engineContext, metadataMetaClient.getActiveTimeline(), - rollbackMetadata, getRecordsGenerationParams(), instantTime, - metadata.getSyncedInstantTime(), wasSynced); - commit(instantTime, records); - closeInternal(); + if (deltacommitsSinceCompaction.containsInstant(deltaCommitInstant)) { + LOG.info("Rolling back MDT deltacommit " + commitInstantTime); + if (!getWriteClient().rollback(commitInstantTime, instantTime)) { + throw new HoodieMetadataException("Failed to rollback deltacommit at " + commitInstantTime); + } + } else { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no corresponding deltacommits on MDT", + commitInstantTime, instantTime)); } + + // Rollback of MOR table may end up adding a new log file. So we need to check for added files and add them to MDT + processAndCommit(instantTime, () -> HoodieTableMetadataUtil.convertMetadataToRecords(engineContext, metadataMetaClient.getActiveTimeline(), Review Comment: Done ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -966,8 +966,8 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { // String syncCommitTime = HoodieTableMetadataUtil.createIndexInitTimestamp(HoodieActiveTimeline.createNewInstantTime()); // TODO: Using METADATA_INDEXER_TIME_SUFFIX for now, but this should have its own suffix. To be fixed after HUDI-6200 String syncCommitTime = HoodieActiveTimeline.createNewInstantTime() + METADATA_INDEXER_TIME_SUFFIX; Review Comment: Done ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -984,52 +984,46 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { */ @Override public void update(HoodieRollbackMetadata rollbackMetadata, String instantTime) { - // The commit which is being rolled back on the dataset - final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); - // Find the deltacommits since the last compaction - Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = - CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); - if (!deltaCommitsInfo.isPresent()) { - LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); - return; - } - - // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) - HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); - HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); - - // The deltacommit that will be rolled back - HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); - - // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions - // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. - if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { - final String compactionInstantTime = compactionInstant.getTimestamp(); - if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { - throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " - + "There are %d deltacommits after this compaction: %s", commitInstantTime, compactionInstantTime, - deltacommitsSinceCompaction.countInstants(), deltacommitsSinceCompaction.getInstants())); + if (initialized && metadata != null) { + // The commit which is being rolled back on the dataset + final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); + // Find the deltacommits since the last compaction + Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = + CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); + if (!deltaCommitsInfo.isPresent() || deltaCommitsInfo.get().getKey().empty()) { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); + return; } - } - if (deltacommitsSinceCompaction.containsInstant(deltaCommitInstant)) { - LOG.info("Rolling back MDT deltacommit " + commitInstantTime); - if (!getWriteClient().rollback(commitInstantTime, instantTime)) { - throw new HoodieMetadataException("Failed to rollback deltacommit at " + commitInstantTime); + // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) + HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); + HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); + + // The deltacommit that will be rolled back + HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); + + // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions Review Comment: latest=oldest to make it clear ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/metadata/HoodieBackedTableMetadataWriter.java: ########## @@ -851,26 +919,49 @@ public void update(HoodieRestoreMetadata restoreMetadata, String instantTime) { */ @Override public void update(HoodieRollbackMetadata rollbackMetadata, String instantTime) { - if (enabled && metadata != null) { - // Is this rollback of an instant that has been synced to the metadata table? - String rollbackInstant = rollbackMetadata.getCommitsRollback().get(0); - boolean wasSynced = metadataMetaClient.getActiveTimeline().containsInstant(new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, rollbackInstant)); - if (!wasSynced) { - // A compaction may have taken place on metadata table which would have included this instant being rolled back. - // Revisit this logic to relax the compaction fencing : https://issues.apache.org/jira/browse/HUDI-2458 - Option<String> latestCompaction = metadata.getLatestCompactionTime(); - if (latestCompaction.isPresent()) { - wasSynced = HoodieTimeline.compareTimestamps(rollbackInstant, HoodieTimeline.LESSER_THAN_OR_EQUALS, latestCompaction.get()); - } + // The commit which is being rolled back on the dataset + final String commitInstantTime = rollbackMetadata.getCommitsRollback().get(0); + // Find the deltacommits since the last compaction + Option<Pair<HoodieTimeline, HoodieInstant>> deltaCommitsInfo = + CompactionUtils.getDeltaCommitsSinceLatestCompaction(metadataMetaClient.getActiveTimeline()); + if (!deltaCommitsInfo.isPresent()) { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no deltacommits on MDT", commitInstantTime, instantTime)); + return; + } + + // This could be a compaction or deltacommit instant (See CompactionUtils.getDeltaCommitsSinceLatestCompaction) + HoodieInstant compactionInstant = deltaCommitsInfo.get().getValue(); + HoodieTimeline deltacommitsSinceCompaction = deltaCommitsInfo.get().getKey(); + + // The deltacommit that will be rolled back + HoodieInstant deltaCommitInstant = new HoodieInstant(false, HoodieTimeline.DELTA_COMMIT_ACTION, commitInstantTime); + + // The commit being rolled back should not be older than the latest compaction on the MDT. Compaction on MDT only occurs when all actions + // are completed on the dataset. Hence, this case implies a rollback of completed commit which should actually be handled using restore. + if (compactionInstant.getAction().equals(HoodieTimeline.COMMIT_ACTION)) { + final String compactionInstantTime = compactionInstant.getTimestamp(); + if (HoodieTimeline.LESSER_THAN_OR_EQUALS.test(commitInstantTime, compactionInstantTime)) { + throw new HoodieMetadataException(String.format("Commit being rolled back %s is older than the latest compaction %s. " + + "There are %d deltacommits after this compaction: %s", commitInstantTime, compactionInstantTime, + deltacommitsSinceCompaction.countInstants(), deltacommitsSinceCompaction.getInstants())); } + } - Map<MetadataPartitionType, HoodieData<HoodieRecord>> records = - HoodieTableMetadataUtil.convertMetadataToRecords(engineContext, metadataMetaClient.getActiveTimeline(), - rollbackMetadata, getRecordsGenerationParams(), instantTime, - metadata.getSyncedInstantTime(), wasSynced); - commit(instantTime, records, false); - closeInternal(); + if (deltaCommitsInfo.get().getKey().containsInstant(deltaCommitInstant)) { + LOG.info("Rolling back MDT deltacommit " + commitInstantTime); + if (!getWriteClient().rollback(commitInstantTime, instantTime)) { + throw new HoodieMetadataException("Failed to rollback deltacommit at " + commitInstantTime); + } + } else { + LOG.info(String.format("Ignoring rollback of instant %s at %s since there are no corresponding deltacommits on MDT", + commitInstantTime, instantTime)); } + + // Rollback of MOR table may end up adding a new log file. So we need to check for added files and add them to MDT + processAndCommit(instantTime, () -> HoodieTableMetadataUtil.convertMetadataToRecords(engineContext, metadataMetaClient.getActiveTimeline(), + rollbackMetadata, getRecordsGenerationParams(), instantTime, + metadata.getSyncedInstantTime(), true), false); Review Comment: I have changed this to use a custom suffix for restore and rollback operations. -- 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: commits-unsubscr...@hudi.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org