You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@iceberg.apache.org by GitBox <gi...@apache.org> on 2022/08/04 21:44:31 UTC

[GitHub] [iceberg] szehon-ho opened a new pull request, #5443: Core: Partition filter pushdown for entries table

szehon-ho opened a new pull request, #5443:
URL: https://github.com/apache/iceberg/pull/5443

   This implements partition filter pushdown for 'entries' and 'all_entries' table. This makes a base class for both and then re-uses the partition filter pushdown logic of 'files' table when doing planning.


-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] szehon-ho commented on pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
szehon-ho commented on PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#issuecomment-1209710530

   Thanks  @rdblue , for the fast review


-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] rdblue commented on a diff in pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
rdblue commented on code in PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#discussion_r939696680


##########
core/src/test/java/org/apache/iceberg/TestMetadataTableFilters.java:
##########
@@ -153,20 +160,46 @@ private boolean isAggFileTable(MetadataTableType tableType) {
     return aggFileTables.contains(tableType);
   }
 
+  private String partitionColumn(String colName) {
+    switch (type) {
+      case FILES:
+      case DATA_FILES:
+      case DELETE_FILES:
+      case ALL_DATA_FILES:
+      case ALL_DELETE_FILES:
+      case ALL_FILES:
+        return String.format("partition.%s", colName);
+      case ENTRIES:
+      case ALL_ENTRIES:
+        return String.format("data_file.partition.%s", colName);
+      default:
+        throw new IllegalArgumentException("Unsupported metadata table type:" + type);
+    }
+  }
+
+  /** @return a basic expression that always evaluates to true, to test AND logic */
+  private Expression dummyExpression() {
+    switch (type) {
+      case FILES:
+      case DATA_FILES:
+      case DELETE_FILES:
+      case ALL_DATA_FILES:
+      case ALL_DELETE_FILES:
+      case ALL_FILES:
+        return Expressions.greaterThan("record_count", 0);
+      case ENTRIES:
+      case ALL_ENTRIES:
+        return Expressions.greaterThan("data_file.record_count", 0);
+      default:
+        throw new IllegalArgumentException("Unsupported metadata table type:" + type);
+    }
+  }
+
   @Test
   public void testNoFilter() {
     Table metadataTable = createMetadataTable();
-    Types.StructType expected =
-        new Schema(
-                required(
-                    102,
-                    "partition",
-                    Types.StructType.of(optional(1000, "data_bucket", Types.IntegerType.get())),
-                    "Partition data tuple, schema based on the partition spec"))
-            .asStruct();
-
-    TableScan scan = metadataTable.newScan().select("partition.data_bucket");
-    Assert.assertEquals(expected, scan.schema().asStruct());
+
+    TableScan scan = metadataTable.newScan().select(partitionColumn("data_bucket"));

Review Comment:
   Why remove the assertion? Is it not useful?



-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] aokolnychyi commented on pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
aokolnychyi commented on PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#issuecomment-1209893192

   LGTM too


-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] szehon-ho merged pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
szehon-ho merged PR #5443:
URL: https://github.com/apache/iceberg/pull/5443


-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] szehon-ho commented on a diff in pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
szehon-ho commented on code in PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#discussion_r940663982


##########
core/src/main/java/org/apache/iceberg/BaseEntriesTable.java:
##########
@@ -0,0 +1,160 @@
+/*
+ * 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 com.github.benmanes.caffeine.cache.Caffeine;
+import com.github.benmanes.caffeine.cache.LoadingCache;
+import java.util.Map;
+import org.apache.iceberg.expressions.Expression;
+import org.apache.iceberg.expressions.Expressions;
+import org.apache.iceberg.expressions.ManifestEvaluator;
+import org.apache.iceberg.expressions.ResidualEvaluator;
+import org.apache.iceberg.io.CloseableIterable;
+import org.apache.iceberg.io.FileIO;
+import org.apache.iceberg.relocated.com.google.common.annotations.VisibleForTesting;
+import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.apache.iceberg.relocated.com.google.common.collect.Maps;
+import org.apache.iceberg.relocated.com.google.common.collect.Sets;
+import org.apache.iceberg.types.Type;
+import org.apache.iceberg.types.TypeUtil;
+import org.apache.iceberg.types.Types.StructType;
+import org.apache.iceberg.util.StructProjection;
+
+/** Base class logic for entries metadata tables */
+abstract class BaseEntriesTable extends BaseMetadataTable {
+
+  BaseEntriesTable(TableOperations ops, Table table, String name) {
+    super(ops, table, name);
+  }
+
+  @Override
+  public Schema schema() {
+    StructType partitionType = Partitioning.partitionType(table());
+    Schema schema = ManifestEntry.getSchema(partitionType);
+    if (partitionType.fields().size() < 1) {
+      // avoid returning an empty struct, which is not always supported. instead, drop the partition
+      // field (id 102)
+      return TypeUtil.selectNot(schema, Sets.newHashSet(102));
+    } else {
+      return schema;
+    }
+  }
+
+  static CloseableIterable<FileScanTask> planFiles(
+      Table table,
+      CloseableIterable<ManifestFile> manifests,
+      Schema tableSchema,
+      Schema projectedSchema,
+      TableScanContext context) {
+    Expression rowFilter = context.rowFilter();
+    boolean caseSensitive = context.caseSensitive();
+    boolean ignoreResiduals = context.ignoreResiduals();
+
+    LoadingCache<Integer, ManifestEvaluator> evalCache =
+        Caffeine.newBuilder()
+            .build(
+                specId -> {
+                  PartitionSpec spec = table.specs().get(specId);
+                  PartitionSpec transformedSpec = BaseFilesTable.transformSpec(tableSchema, spec);
+                  return ManifestEvaluator.forRowFilter(rowFilter, transformedSpec, caseSensitive);
+                });
+
+    CloseableIterable<ManifestFile> filteredManifests =
+        CloseableIterable.filter(
+            manifests, manifest -> evalCache.get(manifest.partitionSpecId()).eval(manifest));
+
+    String schemaString = SchemaParser.toJson(projectedSchema);
+    String specString = PartitionSpecParser.toJson(PartitionSpec.unpartitioned());
+    Expression filter = ignoreResiduals ? Expressions.alwaysTrue() : rowFilter;
+    ResidualEvaluator residuals = ResidualEvaluator.unpartitioned(filter);
+
+    return CloseableIterable.transform(
+        filteredManifests,
+        manifest ->
+            new ManifestReadTask(
+                table, manifest, projectedSchema, schemaString, specString, residuals));
+  }
+
+  @Override
+  MetadataTableType metadataTableType() {
+    return MetadataTableType.ALL_ENTRIES;

Review Comment:
   Whoops you are right, thanks



##########
core/src/main/java/org/apache/iceberg/BaseEntriesTable.java:
##########
@@ -0,0 +1,160 @@
+/*
+ * 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 com.github.benmanes.caffeine.cache.Caffeine;
+import com.github.benmanes.caffeine.cache.LoadingCache;
+import java.util.Map;
+import org.apache.iceberg.expressions.Expression;
+import org.apache.iceberg.expressions.Expressions;
+import org.apache.iceberg.expressions.ManifestEvaluator;
+import org.apache.iceberg.expressions.ResidualEvaluator;
+import org.apache.iceberg.io.CloseableIterable;
+import org.apache.iceberg.io.FileIO;
+import org.apache.iceberg.relocated.com.google.common.annotations.VisibleForTesting;
+import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.apache.iceberg.relocated.com.google.common.collect.Maps;
+import org.apache.iceberg.relocated.com.google.common.collect.Sets;
+import org.apache.iceberg.types.Type;
+import org.apache.iceberg.types.TypeUtil;
+import org.apache.iceberg.types.Types.StructType;
+import org.apache.iceberg.util.StructProjection;
+
+/** Base class logic for entries metadata tables */
+abstract class BaseEntriesTable extends BaseMetadataTable {
+
+  BaseEntriesTable(TableOperations ops, Table table, String name) {
+    super(ops, table, name);
+  }
+
+  @Override
+  public Schema schema() {
+    StructType partitionType = Partitioning.partitionType(table());
+    Schema schema = ManifestEntry.getSchema(partitionType);
+    if (partitionType.fields().size() < 1) {
+      // avoid returning an empty struct, which is not always supported. instead, drop the partition
+      // field (id 102)
+      return TypeUtil.selectNot(schema, Sets.newHashSet(102));

Review Comment:
   Good catch, done



-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] rdblue commented on pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
rdblue commented on PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#issuecomment-1207448238

   Looks good overall, just a couple of minor things you may want to look at.


-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] rdblue commented on a diff in pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
rdblue commented on code in PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#discussion_r939696067


##########
core/src/main/java/org/apache/iceberg/BaseEntriesTable.java:
##########
@@ -0,0 +1,160 @@
+/*
+ * 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 com.github.benmanes.caffeine.cache.Caffeine;
+import com.github.benmanes.caffeine.cache.LoadingCache;
+import java.util.Map;
+import org.apache.iceberg.expressions.Expression;
+import org.apache.iceberg.expressions.Expressions;
+import org.apache.iceberg.expressions.ManifestEvaluator;
+import org.apache.iceberg.expressions.ResidualEvaluator;
+import org.apache.iceberg.io.CloseableIterable;
+import org.apache.iceberg.io.FileIO;
+import org.apache.iceberg.relocated.com.google.common.annotations.VisibleForTesting;
+import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.apache.iceberg.relocated.com.google.common.collect.Maps;
+import org.apache.iceberg.relocated.com.google.common.collect.Sets;
+import org.apache.iceberg.types.Type;
+import org.apache.iceberg.types.TypeUtil;
+import org.apache.iceberg.types.Types.StructType;
+import org.apache.iceberg.util.StructProjection;
+
+/** Base class logic for entries metadata tables */
+abstract class BaseEntriesTable extends BaseMetadataTable {
+
+  BaseEntriesTable(TableOperations ops, Table table, String name) {
+    super(ops, table, name);
+  }
+
+  @Override
+  public Schema schema() {
+    StructType partitionType = Partitioning.partitionType(table());
+    Schema schema = ManifestEntry.getSchema(partitionType);
+    if (partitionType.fields().size() < 1) {
+      // avoid returning an empty struct, which is not always supported. instead, drop the partition
+      // field (id 102)
+      return TypeUtil.selectNot(schema, Sets.newHashSet(102));
+    } else {
+      return schema;
+    }
+  }
+
+  static CloseableIterable<FileScanTask> planFiles(
+      Table table,
+      CloseableIterable<ManifestFile> manifests,
+      Schema tableSchema,
+      Schema projectedSchema,
+      TableScanContext context) {
+    Expression rowFilter = context.rowFilter();
+    boolean caseSensitive = context.caseSensitive();
+    boolean ignoreResiduals = context.ignoreResiduals();
+
+    LoadingCache<Integer, ManifestEvaluator> evalCache =
+        Caffeine.newBuilder()
+            .build(
+                specId -> {
+                  PartitionSpec spec = table.specs().get(specId);
+                  PartitionSpec transformedSpec = BaseFilesTable.transformSpec(tableSchema, spec);
+                  return ManifestEvaluator.forRowFilter(rowFilter, transformedSpec, caseSensitive);
+                });
+
+    CloseableIterable<ManifestFile> filteredManifests =
+        CloseableIterable.filter(
+            manifests, manifest -> evalCache.get(manifest.partitionSpecId()).eval(manifest));
+
+    String schemaString = SchemaParser.toJson(projectedSchema);
+    String specString = PartitionSpecParser.toJson(PartitionSpec.unpartitioned());
+    Expression filter = ignoreResiduals ? Expressions.alwaysTrue() : rowFilter;
+    ResidualEvaluator residuals = ResidualEvaluator.unpartitioned(filter);
+
+    return CloseableIterable.transform(
+        filteredManifests,
+        manifest ->
+            new ManifestReadTask(
+                table, manifest, projectedSchema, schemaString, specString, residuals));
+  }
+
+  @Override
+  MetadataTableType metadataTableType() {
+    return MetadataTableType.ALL_ENTRIES;

Review Comment:
   This should be abstract, right?



-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] rdblue commented on a diff in pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
rdblue commented on code in PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#discussion_r939695967


##########
core/src/main/java/org/apache/iceberg/BaseEntriesTable.java:
##########
@@ -0,0 +1,160 @@
+/*
+ * 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 com.github.benmanes.caffeine.cache.Caffeine;
+import com.github.benmanes.caffeine.cache.LoadingCache;
+import java.util.Map;
+import org.apache.iceberg.expressions.Expression;
+import org.apache.iceberg.expressions.Expressions;
+import org.apache.iceberg.expressions.ManifestEvaluator;
+import org.apache.iceberg.expressions.ResidualEvaluator;
+import org.apache.iceberg.io.CloseableIterable;
+import org.apache.iceberg.io.FileIO;
+import org.apache.iceberg.relocated.com.google.common.annotations.VisibleForTesting;
+import org.apache.iceberg.relocated.com.google.common.collect.ImmutableList;
+import org.apache.iceberg.relocated.com.google.common.collect.Maps;
+import org.apache.iceberg.relocated.com.google.common.collect.Sets;
+import org.apache.iceberg.types.Type;
+import org.apache.iceberg.types.TypeUtil;
+import org.apache.iceberg.types.Types.StructType;
+import org.apache.iceberg.util.StructProjection;
+
+/** Base class logic for entries metadata tables */
+abstract class BaseEntriesTable extends BaseMetadataTable {
+
+  BaseEntriesTable(TableOperations ops, Table table, String name) {
+    super(ops, table, name);
+  }
+
+  @Override
+  public Schema schema() {
+    StructType partitionType = Partitioning.partitionType(table());
+    Schema schema = ManifestEntry.getSchema(partitionType);
+    if (partitionType.fields().size() < 1) {
+      // avoid returning an empty struct, which is not always supported. instead, drop the partition
+      // field (id 102)
+      return TypeUtil.selectNot(schema, Sets.newHashSet(102));

Review Comment:
   Can you use `DataFile.PARTITION_ID` here? We hard-coded 102 before that was a constant.



-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org


[GitHub] [iceberg] szehon-ho commented on a diff in pull request #5443: Core: Partition filter pushdown for entries table

Posted by GitBox <gi...@apache.org>.
szehon-ho commented on code in PR #5443:
URL: https://github.com/apache/iceberg/pull/5443#discussion_r940665416


##########
core/src/test/java/org/apache/iceberg/TestMetadataTableFilters.java:
##########
@@ -153,20 +160,46 @@ private boolean isAggFileTable(MetadataTableType tableType) {
     return aggFileTables.contains(tableType);
   }
 
+  private String partitionColumn(String colName) {
+    switch (type) {
+      case FILES:
+      case DATA_FILES:
+      case DELETE_FILES:
+      case ALL_DATA_FILES:
+      case ALL_DELETE_FILES:
+      case ALL_FILES:
+        return String.format("partition.%s", colName);
+      case ENTRIES:
+      case ALL_ENTRIES:
+        return String.format("data_file.partition.%s", colName);
+      default:
+        throw new IllegalArgumentException("Unsupported metadata table type:" + type);
+    }
+  }
+
+  /** @return a basic expression that always evaluates to true, to test AND logic */
+  private Expression dummyExpression() {
+    switch (type) {
+      case FILES:
+      case DATA_FILES:
+      case DELETE_FILES:
+      case ALL_DATA_FILES:
+      case ALL_DELETE_FILES:
+      case ALL_FILES:
+        return Expressions.greaterThan("record_count", 0);
+      case ENTRIES:
+      case ALL_ENTRIES:
+        return Expressions.greaterThan("data_file.record_count", 0);
+      default:
+        throw new IllegalArgumentException("Unsupported metadata table type:" + type);
+    }
+  }
+
   @Test
   public void testNoFilter() {
     Table metadataTable = createMetadataTable();
-    Types.StructType expected =
-        new Schema(
-                required(
-                    102,
-                    "partition",
-                    Types.StructType.of(optional(1000, "data_bucket", Types.IntegerType.get())),
-                    "Partition data tuple, schema based on the partition spec"))
-            .asStruct();
-
-    TableScan scan = metadataTable.newScan().select("partition.data_bucket");
-    Assert.assertEquals(expected, scan.schema().asStruct());
+
+    TableScan scan = metadataTable.newScan().select(partitionColumn("data_bucket"));

Review Comment:
   Ah I wrote this Test class initially to have one partition-filter-pushdown test various metadata tables (files, all_files, etc), but this assertion would kind of not work here.  There should be other tests that assert the same thing (can add if not), and also end to end sql tests that excercise this.



-- 
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-unsubscribe@iceberg.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscribe@iceberg.apache.org
For additional commands, e-mail: issues-help@iceberg.apache.org