Skip to content
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

HIVE-28276: Iceberg:Make Iceberg split threads configurable when table scanning #5260

Merged
merged 5 commits into from
Jun 3, 2024
Merged
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,7 @@
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import java.util.concurrent.ExecutorService;
import java.util.function.BiFunction;
import java.util.stream.Collectors;
import java.util.stream.Stream;
Expand Down Expand Up @@ -62,6 +63,7 @@
import org.apache.iceberg.SchemaParser;
import org.apache.iceberg.SnapshotRef;
import org.apache.iceberg.StructLike;
import org.apache.iceberg.SystemConfigs;
import org.apache.iceberg.Table;
import org.apache.iceberg.TableProperties;
import org.apache.iceberg.TableScan;
Expand Down Expand Up @@ -97,6 +99,7 @@
import org.apache.iceberg.types.Types;
import org.apache.iceberg.util.PartitionUtil;
import org.apache.iceberg.util.SerializationUtil;
import org.apache.iceberg.util.ThreadPools;

/**
* Generic Mrv2 InputFormat API for Iceberg.
Expand Down Expand Up @@ -207,19 +210,30 @@ public List<InputSplit> getSplits(JobContext context) {
conf.set(InputFormatConfig.SERIALIZED_TABLE_PREFIX + tbl.name(), SerializationUtil.serializeToBase64(tbl));
return tbl;
});
final ExecutorService workerPool =
ThreadPools.newWorkerPool("iceberg-plan-worker-pool",
conf.getInt(SystemConfigs.WORKER_THREAD_POOL_SIZE.propertyKey(), ThreadPools.WORKER_THREAD_POOL_SIZE));
try {
return planInputSplits(table, conf, workerPool);
} finally {
workerPool.shutdown();
}
}

private List<InputSplit> planInputSplits(Table table, Configuration conf, ExecutorService workerPool) {
List<InputSplit> splits = Lists.newArrayList();
boolean applyResidual = !conf.getBoolean(InputFormatConfig.SKIP_RESIDUAL_FILTERING, false);
InputFormatConfig.InMemoryDataModel model = conf.getEnum(InputFormatConfig.IN_MEMORY_DATA_MODEL,
InputFormatConfig.InMemoryDataModel.GENERIC);

long fromVersion = conf.getLong(InputFormatConfig.SNAPSHOT_ID_INTERVAL_FROM, -1);
Scan<?, FileScanTask, CombinedScanTask> scan;
Scan<? extends Scan, FileScanTask, CombinedScanTask> scan;
if (fromVersion != -1) {
scan = applyConfig(conf, createIncrementalAppendScan(table, conf));
} else {
scan = applyConfig(conf, createTableScan(table, conf));
}
scan = scan.planWith(workerPool);

boolean allowDataFilesWithinTableLocationOnly =
conf.getBoolean(HiveConf.ConfVars.HIVE_ICEBERG_ALLOW_DATAFILES_IN_TABLE_LOCATION_ONLY.varname,
Expand Down
Loading