-
Notifications
You must be signed in to change notification settings - Fork 3.1k
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
Use dynamic filter to prune Iceberg splits based on partition values #9193
Merged
findepi
merged 5 commits into
trinodb:master
from
alexjo2144:iceberg/dynamic-filter-partitions
Oct 25, 2021
Merged
Changes from all commits
Commits
Show all changes
5 commits
Select commit
Hold shift + click to select a range
237137a
Verify query correctness in TestHiveDynamicPartitionPruning
alexjo2144 1b231c7
Add AbstractTestDynamicPartitionPruning test class
alexjo2144 0fb2b31
Add back test for Hive dynamic filtering of partition
alexjo2144 0924252
Use dynamic filter to prune Iceberg splits
alexjo2144 2bb6095
empty
findepi File filter
Filter by extension
Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
There are no files selected for viewing
50 changes: 50 additions & 0 deletions
50
...in/trino-hive/src/test/java/io/trino/plugin/hive/TestHiveDynamicPartitionPruningTest.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,50 @@ | ||
/* | ||
* 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.hive; | ||
|
||
import com.google.common.collect.ImmutableMap; | ||
import io.trino.testing.BaseDynamicPartitionPruningTest; | ||
import io.trino.testing.QueryRunner; | ||
import org.intellij.lang.annotations.Language; | ||
|
||
import java.util.List; | ||
|
||
import static java.lang.String.format; | ||
import static java.util.stream.Collectors.joining; | ||
|
||
public class TestHiveDynamicPartitionPruningTest | ||
extends BaseDynamicPartitionPruningTest | ||
{ | ||
@Override | ||
protected QueryRunner createQueryRunner() | ||
throws Exception | ||
{ | ||
return HiveQueryRunner.builder() | ||
.setExtraProperties(EXTRA_PROPERTIES) | ||
.setHiveProperties(ImmutableMap.of("hive.dynamic-filtering-probe-blocking-timeout", "1h")) | ||
.setInitialTables(REQUIRED_TABLES) | ||
.build(); | ||
} | ||
|
||
@Override | ||
protected void createLineitemTable(String tableName, List<String> columns, List<String> partitionColumns) | ||
{ | ||
@Language("SQL") String sql = format( | ||
"CREATE TABLE %s WITH (format = 'TEXTFILE', partitioned_by=array[%s]) AS SELECT %s FROM tpch.tiny.lineitem", | ||
tableName, | ||
partitionColumns.stream().map(column -> "'" + column + "'").collect(joining(",")), | ||
String.join(",", columns)); | ||
getQueryRunner().execute(sql); | ||
} | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -14,6 +14,7 @@ | |
package io.trino.plugin.iceberg; | ||
|
||
import com.google.common.collect.ImmutableList; | ||
import io.airlift.units.Duration; | ||
import io.trino.plugin.base.classloader.ClassLoaderSafeConnectorSplitSource; | ||
import io.trino.spi.connector.ConnectorSession; | ||
import io.trino.spi.connector.ConnectorSplitManager; | ||
|
@@ -22,12 +23,19 @@ | |
import io.trino.spi.connector.ConnectorTransactionHandle; | ||
import io.trino.spi.connector.DynamicFilter; | ||
import io.trino.spi.connector.FixedSplitSource; | ||
import io.trino.spi.type.TypeManager; | ||
import org.apache.iceberg.PartitionField; | ||
import org.apache.iceberg.Table; | ||
import org.apache.iceberg.TableScan; | ||
|
||
import javax.inject.Inject; | ||
|
||
import static io.trino.plugin.iceberg.ExpressionConverter.toIcebergExpression; | ||
import java.util.Set; | ||
|
||
import static com.google.common.collect.ImmutableSet.toImmutableSet; | ||
import static io.trino.plugin.iceberg.IcebergSessionProperties.getDynamicFilteringWaitTimeout; | ||
import static io.trino.plugin.iceberg.IcebergUtil.getColumns; | ||
import static io.trino.plugin.iceberg.IcebergUtil.getIdentityPartitions; | ||
import static java.util.Objects.requireNonNull; | ||
|
||
public class IcebergSplitManager | ||
|
@@ -36,11 +44,13 @@ public class IcebergSplitManager | |
public static final int ICEBERG_DOMAIN_COMPACTION_THRESHOLD = 1000; | ||
|
||
private final IcebergTransactionManager transactionManager; | ||
private final TypeManager typeManager; | ||
|
||
@Inject | ||
public IcebergSplitManager(IcebergTransactionManager transactionManager) | ||
public IcebergSplitManager(IcebergTransactionManager transactionManager, TypeManager typeManager) | ||
{ | ||
this.transactionManager = requireNonNull(transactionManager, "transactionManager is null"); | ||
this.typeManager = requireNonNull(typeManager, "typeManager is null"); | ||
} | ||
|
||
@Override | ||
|
@@ -58,17 +68,24 @@ public ConnectorSplitSource getSplits( | |
} | ||
|
||
Table icebergTable = transactionManager.get(transaction).getIcebergTable(session, table.getSchemaTableName()); | ||
Duration dynamicFilteringWaitTimeout = getDynamicFilteringWaitTimeout(session); | ||
|
||
Set<Integer> identityPartitionFieldIds = getIdentityPartitions(icebergTable.spec()).keySet().stream() | ||
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. This is also called in don't change anything here though |
||
.map(PartitionField::sourceId) | ||
.collect(toImmutableSet()); | ||
Set<IcebergColumnHandle> identityPartitionColumns = getColumns(icebergTable.schema(), typeManager).stream() | ||
.filter(column -> identityPartitionFieldIds.contains(column.getId())) | ||
.collect(toImmutableSet()); | ||
|
||
TableScan tableScan = icebergTable.newScan() | ||
.filter(toIcebergExpression( | ||
table.getEnforcedPredicate() | ||
// TODO: Remove TupleDomain#simplify once Iceberg supports IN expression. Currently this | ||
// is required for IN predicates on non-partition columns with large value list. Such | ||
// predicates on partition columns are not supported. | ||
// (See AbstractTestIcebergSmoke#testLargeInFailureOnPartitionedColumns) | ||
.intersect(table.getUnenforcedPredicate().simplify(ICEBERG_DOMAIN_COMPACTION_THRESHOLD)))) | ||
.useSnapshot(table.getSnapshotId().get()); | ||
IcebergSplitSource splitSource = new IcebergSplitSource(table.getSchemaTableName(), tableScan.planTasks()); | ||
IcebergSplitSource splitSource = new IcebergSplitSource( | ||
table, | ||
identityPartitionColumns, | ||
tableScan, | ||
dynamicFilter, | ||
session.getTimeZoneKey(), | ||
dynamicFilteringWaitTimeout); | ||
|
||
return new ClassLoaderSafeConnectorSplitSource(splitSource, Thread.currentThread().getContextClassLoader()); | ||
} | ||
|
Oops, something went wrong.
Add this suggestion to a batch that can be applied as a single commit.
This suggestion is invalid because no changes were made to the code.
Suggestions cannot be applied while the pull request is closed.
Suggestions cannot be applied while viewing a subset of changes.
Only one suggestion per line can be applied in a batch.
Add this suggestion to a batch that can be applied as a single commit.
Applying suggestions on deleted lines is not supported.
You must change the existing code in this line in order to create a valid suggestion.
Outdated suggestions cannot be applied.
This suggestion has been applied or marked resolved.
Suggestions cannot be applied from pending reviews.
Suggestions cannot be applied on multi-line comments.
Suggestions cannot be applied while the pull request is queued to merge.
Suggestion cannot be applied right now. Please check back later.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
In hive it's called
hive.dynamic-filtering-probe-blocking-timeout
. idk what "probe" is supposed to mean ("probe side"? that's engine level concept, not connectors').i like the name here better, but we may want to align the two (can be follow up)
cc @sopel39 @raunaqmorarka @losipiuk
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
i see @raunaqmorarka ACKd this in #9193 (comment)
@raunaqmorarka can you please follow-up with rename in Hive?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
yup, will do
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
#9751