aokolnychyi commented on a change in pull request #2564: URL: https://github.com/apache/iceberg/pull/2564#discussion_r633697226
########## File path: core/src/main/java/org/apache/iceberg/ReachableFileUtil.java ########## @@ -0,0 +1,114 @@ +/* + * 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; + +import java.util.List; +import java.util.Set; +import java.util.stream.Collectors; +import org.apache.iceberg.hadoop.Util; +import org.apache.iceberg.io.FileIO; +import org.apache.iceberg.relocated.com.google.common.collect.Lists; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class ReachableFileUtil { + + private static final Logger LOG = LoggerFactory.getLogger(ReachableFileUtil.class); + + private ReachableFileUtil() { + } + + /** + * Returns the location of the version hint file + * + * @param table table for which version hint file's path needs to be retrieved + * @return the location of the version hint file + */ + public static String versionHintLocation(Table table) { + TableOperations ops = ((HasTableOperations) table).operations(); + return ops.metadataFileLocation(Util.VERSION_HINT_FILENAME); + } + + /** + * Returns locations of JSON metadata files in a table. + * + * @param table Table to get JSON metadata files from + * @param recursive When true, recursively retrieves all the reachable JSON metadata files. + * When false, gets the all the JSON metadata files only from the current metadata. + * @return locations of JSON metadata files + */ + public static Set<String> metadataFileLocations(Table table, boolean recursive) { + Set<String> metadataFileLocations = Sets.newHashSet(); + TableOperations ops = ((HasTableOperations) table).operations(); + TableMetadata tableMetadata = ops.current(); + metadataFileLocations.add(tableMetadata.metadataFileLocation()); + metadataFileLocations(tableMetadata, metadataFileLocations, ops.io(), recursive); + return metadataFileLocations; + } + + private static void metadataFileLocations(TableMetadata metadata, Set<String> metadataFileLocations, + FileIO io, boolean recursive) { + List<TableMetadata.MetadataLogEntry> metadataLogEntries = metadata.previousFiles(); + if (metadataLogEntries.size() > 0) { + List<String> metadataLocations = metadataLogEntries.stream() + .map(TableMetadata.MetadataLogEntry::file) + .collect(Collectors.toList()); + metadataFileLocations.addAll(metadataLocations); + if (recursive) { + TableMetadata previousMetadata = findFirstExistentPreviousMetadata(metadataLocations, io); + if (previousMetadata != null) { + metadataFileLocations(previousMetadata, metadataFileLocations, io, recursive); + } + } + } + } + + private static TableMetadata findFirstExistentPreviousMetadata(List<String> metadataLocations, FileIO io) { + TableMetadata metadata = null; + for (String metadataLocation : metadataLocations) { + try { + metadata = TableMetadataParser.read(io, metadataLocation); Review comment: nit: Extra space before `TableMetadataParser` ########## File path: core/src/main/java/org/apache/iceberg/ReachableFileUtil.java ########## @@ -0,0 +1,114 @@ +/* + * 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; + +import java.util.List; +import java.util.Set; +import java.util.stream.Collectors; +import org.apache.iceberg.hadoop.Util; +import org.apache.iceberg.io.FileIO; +import org.apache.iceberg.relocated.com.google.common.collect.Lists; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class ReachableFileUtil { + + private static final Logger LOG = LoggerFactory.getLogger(ReachableFileUtil.class); + + private ReachableFileUtil() { + } + + /** + * Returns the location of the version hint file + * + * @param table table for which version hint file's path needs to be retrieved + * @return the location of the version hint file + */ + public static String versionHintLocation(Table table) { + TableOperations ops = ((HasTableOperations) table).operations(); + return ops.metadataFileLocation(Util.VERSION_HINT_FILENAME); + } + + /** + * Returns locations of JSON metadata files in a table. + * + * @param table Table to get JSON metadata files from + * @param recursive When true, recursively retrieves all the reachable JSON metadata files. + * When false, gets the all the JSON metadata files only from the current metadata. + * @return locations of JSON metadata files + */ + public static Set<String> metadataFileLocations(Table table, boolean recursive) { + Set<String> metadataFileLocations = Sets.newHashSet(); + TableOperations ops = ((HasTableOperations) table).operations(); + TableMetadata tableMetadata = ops.current(); + metadataFileLocations.add(tableMetadata.metadataFileLocation()); + metadataFileLocations(tableMetadata, metadataFileLocations, ops.io(), recursive); + return metadataFileLocations; + } + + private static void metadataFileLocations(TableMetadata metadata, Set<String> metadataFileLocations, + FileIO io, boolean recursive) { + List<TableMetadata.MetadataLogEntry> metadataLogEntries = metadata.previousFiles(); Review comment: nit: let's import `MetadataLogEntry` directly to shorten the lines. ########## File path: core/src/main/java/org/apache/iceberg/ReachableFileUtil.java ########## @@ -0,0 +1,114 @@ +/* + * 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; + +import java.util.List; +import java.util.Set; +import java.util.stream.Collectors; +import org.apache.iceberg.hadoop.Util; +import org.apache.iceberg.io.FileIO; +import org.apache.iceberg.relocated.com.google.common.collect.Lists; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +public class ReachableFileUtil { + + private static final Logger LOG = LoggerFactory.getLogger(ReachableFileUtil.class); + + private ReachableFileUtil() { + } + + /** + * Returns the location of the version hint file + * + * @param table table for which version hint file's path needs to be retrieved + * @return the location of the version hint file + */ + public static String versionHintLocation(Table table) { + TableOperations ops = ((HasTableOperations) table).operations(); + return ops.metadataFileLocation(Util.VERSION_HINT_FILENAME); + } + + /** + * Returns locations of JSON metadata files in a table. + * + * @param table Table to get JSON metadata files from + * @param recursive When true, recursively retrieves all the reachable JSON metadata files. + * When false, gets the all the JSON metadata files only from the current metadata. + * @return locations of JSON metadata files + */ + public static Set<String> metadataFileLocations(Table table, boolean recursive) { + Set<String> metadataFileLocations = Sets.newHashSet(); + TableOperations ops = ((HasTableOperations) table).operations(); + TableMetadata tableMetadata = ops.current(); + metadataFileLocations.add(tableMetadata.metadataFileLocation()); + metadataFileLocations(tableMetadata, metadataFileLocations, ops.io(), recursive); + return metadataFileLocations; + } + + private static void metadataFileLocations(TableMetadata metadata, Set<String> metadataFileLocations, + FileIO io, boolean recursive) { + List<TableMetadata.MetadataLogEntry> metadataLogEntries = metadata.previousFiles(); + if (metadataLogEntries.size() > 0) { + List<String> metadataLocations = metadataLogEntries.stream() Review comment: nit: I think it would be best to avoid the temp list here as this method maybe called in a memory-sensitive env. I liked the previous implementation when we just iterated through `MetadataLogEntry`. You can also pass log entries to the method below. ########## File path: spark/src/main/java/org/apache/iceberg/spark/actions/BaseSparkAction.java ########## @@ -167,19 +133,21 @@ protected Table newStaticTable(TableMetadata metadata, FileIO io) { } protected Dataset<Row> buildManifestListDF(Table table) { - List<String> manifestLists = getManifestListPaths(table.snapshots()); + List<String> manifestLists = ReachableFileUtil.manifestListLocations(table); return spark.createDataset(manifestLists, Encoders.STRING()).toDF("file_path"); } - protected Dataset<Row> buildOtherMetadataFileDF(TableOperations ops) { - List<String> otherMetadataFiles = getOtherMetadataFilePaths(ops); - return spark.createDataset(otherMetadataFiles, Encoders.STRING()).toDF("file_path"); + protected Dataset<Row> buildOtherMetadataFileDF(Table table) { + Set<String> otherMetadataFiles = ReachableFileUtil.metadataFileLocations(table, false); Review comment: nit: I think it is a little confusing to call metadata file locations as otherMetadataFiles and then add version hint to it. Plus, the set returned from the utility may be immutable in the future. What about this? ``` List<String> otherMetadataFiles = Lists.newArrayList(); otherMetadataFiles.addAll(ReachableFileUtil.metadataFileLocations(table, false)); otherMetadataFiles.add(ReachableFileUtil.versionHintLocation(table)); return spark.createDataset(otherMetadataFiles, Encoders.STRING()).toDF("file_path"); ``` -- 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. For queries about this service, please contact Infrastructure at: [email protected] --------------------------------------------------------------------- To unsubscribe, e-mail: [email protected] For additional commands, e-mail: [email protected]
