|
19 | 19 | package org.apache.paimon.flink.action; |
20 | 20 |
|
21 | 21 | import org.apache.paimon.CoreOptions; |
| 22 | +import org.apache.paimon.append.cluster.IncrementalClusterManager; |
| 23 | +import org.apache.paimon.compact.CompactUnit; |
22 | 24 | import org.apache.paimon.data.BinaryRow; |
23 | 25 | import org.apache.paimon.data.InternalRow; |
24 | 26 | import org.apache.paimon.flink.FlinkConnectorOptions; |
| 27 | +import org.apache.paimon.flink.cluster.IncrementalClusterSplitSource; |
| 28 | +import org.apache.paimon.flink.cluster.RewriteIncrementalClusterCommittableOperator; |
25 | 29 | import org.apache.paimon.flink.compact.AppendTableCompactBuilder; |
26 | 30 | import org.apache.paimon.flink.postpone.PostponeBucketCompactSplitSource; |
27 | 31 | import org.apache.paimon.flink.postpone.RewritePostponeBucketCommittableOperator; |
|
32 | 36 | import org.apache.paimon.flink.sink.FixedBucketSink; |
33 | 37 | import org.apache.paimon.flink.sink.FlinkSinkBuilder; |
34 | 38 | import org.apache.paimon.flink.sink.FlinkStreamPartitioner; |
| 39 | +import org.apache.paimon.flink.sink.RowAppendTableSink; |
35 | 40 | import org.apache.paimon.flink.sink.RowDataChannelComputer; |
| 41 | +import org.apache.paimon.flink.sorter.TableSortInfo; |
| 42 | +import org.apache.paimon.flink.sorter.TableSorter; |
36 | 43 | import org.apache.paimon.flink.source.CompactorSourceBuilder; |
37 | 44 | import org.apache.paimon.manifest.ManifestEntry; |
38 | 45 | import org.apache.paimon.options.Options; |
|
43 | 50 | import org.apache.paimon.predicate.PredicateProjectionConverter; |
44 | 51 | import org.apache.paimon.table.BucketMode; |
45 | 52 | import org.apache.paimon.table.FileStoreTable; |
| 53 | +import org.apache.paimon.table.source.DataSplit; |
46 | 54 | import org.apache.paimon.types.RowType; |
47 | 55 | import org.apache.paimon.utils.InternalRowPartitionComputer; |
48 | 56 | import org.apache.paimon.utils.Pair; |
|
67 | 75 | import java.util.LinkedHashMap; |
68 | 76 | import java.util.List; |
69 | 77 | import java.util.Map; |
| 78 | +import java.util.stream.Collectors; |
70 | 79 |
|
71 | 80 | import static org.apache.paimon.partition.PartitionPredicate.createBinaryPartitions; |
72 | 81 | import static org.apache.paimon.partition.PartitionPredicate.createPartitionPredicate; |
@@ -100,10 +109,6 @@ public CompactAction( |
100 | 109 | checkArgument( |
101 | 110 | !((FileStoreTable) table).coreOptions().dataEvolutionEnabled(), |
102 | 111 | "Compact action does not support data evolution table yet. "); |
103 | | - checkArgument( |
104 | | - !(((FileStoreTable) table).bucketMode() == BucketMode.BUCKET_UNAWARE |
105 | | - && ((FileStoreTable) table).coreOptions().clusteringIncrementalEnabled()), |
106 | | - "The table has enabled incremental clustering, and do not support compact in flink yet."); |
107 | 112 | HashMap<String, String> dynamicOptions = new HashMap<>(tableConf); |
108 | 113 | dynamicOptions.put(CoreOptions.WRITE_ONLY.key(), "false"); |
109 | 114 | table = table.copy(dynamicOptions); |
@@ -147,8 +152,12 @@ private boolean buildImpl() throws Exception { |
147 | 152 | if (fileStoreTable.coreOptions().bucket() == BucketMode.POSTPONE_BUCKET) { |
148 | 153 | return buildForPostponeBucketCompaction(env, fileStoreTable, isStreaming); |
149 | 154 | } else if (fileStoreTable.bucketMode() == BucketMode.BUCKET_UNAWARE) { |
150 | | - buildForAppendTableCompact(env, fileStoreTable, isStreaming); |
151 | | - return true; |
| 155 | + if (fileStoreTable.coreOptions().clusteringIncrementalEnabled()) { |
| 156 | + return buildForIncrementalClustering(env, fileStoreTable, isStreaming); |
| 157 | + } else { |
| 158 | + buildForAppendTableCompact(env, fileStoreTable, isStreaming); |
| 159 | + return true; |
| 160 | + } |
152 | 161 | } else { |
153 | 162 | buildForBucketedTableCompact(env, fileStoreTable, isStreaming); |
154 | 163 | return true; |
@@ -202,6 +211,138 @@ private void buildForAppendTableCompact( |
202 | 211 | builder.build(); |
203 | 212 | } |
204 | 213 |
|
| 214 | + private boolean buildForIncrementalClustering( |
| 215 | + StreamExecutionEnvironment env, FileStoreTable table, boolean isStreaming) { |
| 216 | + checkArgument(!isStreaming, "Incremental clustering currently only supports batch mode"); |
| 217 | + checkArgument( |
| 218 | + partitions == null, |
| 219 | + "Incremental clustering currently does not support specifying partitions"); |
| 220 | + checkArgument( |
| 221 | + whereSql == null, "Incremental clustering currently does not support predicates"); |
| 222 | + |
| 223 | + IncrementalClusterManager incrementalClusterManager = new IncrementalClusterManager(table); |
| 224 | + |
| 225 | + // non-full strategy as default for incremental clustering |
| 226 | + if (fullCompaction == null) { |
| 227 | + fullCompaction = false; |
| 228 | + } |
| 229 | + Options options = new Options(table.options()); |
| 230 | + int localSampleMagnification = table.coreOptions().getLocalSampleMagnification(); |
| 231 | + if (localSampleMagnification < 20) { |
| 232 | + throw new IllegalArgumentException( |
| 233 | + String.format( |
| 234 | + "the config '%s=%d' should not be set too small, greater than or equal to 20 is needed.", |
| 235 | + CoreOptions.SORT_COMPACTION_SAMPLE_MAGNIFICATION.key(), |
| 236 | + localSampleMagnification)); |
| 237 | + } |
| 238 | + String commitUser = CoreOptions.createCommitUser(options); |
| 239 | + InternalRowPartitionComputer partitionComputer = |
| 240 | + new InternalRowPartitionComputer( |
| 241 | + table.coreOptions().partitionDefaultName(), |
| 242 | + table.store().partitionType(), |
| 243 | + table.partitionKeys().toArray(new String[0]), |
| 244 | + table.coreOptions().legacyPartitionName()); |
| 245 | + |
| 246 | + // 1. pick cluster files for each partition |
| 247 | + Map<BinaryRow, CompactUnit> compactUnits = |
| 248 | + incrementalClusterManager.prepareForCluster(fullCompaction); |
| 249 | + if (compactUnits.isEmpty()) { |
| 250 | + LOGGER.info( |
| 251 | + "No partition needs to be incrementally clustered. " |
| 252 | + + "Please set '--compact_strategy full' if you need to forcibly trigger the cluster."); |
| 253 | + if (this.forceStartFlinkJob) { |
| 254 | + env.fromSequence(0, 0) |
| 255 | + .name("Nothing to Cluster Source") |
| 256 | + .sinkTo(new DiscardingSink<>()); |
| 257 | + return true; |
| 258 | + } else { |
| 259 | + return false; |
| 260 | + } |
| 261 | + } |
| 262 | + Map<BinaryRow, DataSplit[]> partitionSplits = |
| 263 | + compactUnits.entrySet().stream() |
| 264 | + .collect( |
| 265 | + Collectors.toMap( |
| 266 | + Map.Entry::getKey, |
| 267 | + entry -> |
| 268 | + incrementalClusterManager |
| 269 | + .toSplits( |
| 270 | + entry.getKey(), |
| 271 | + entry.getValue().files()) |
| 272 | + .toArray(new DataSplit[0]))); |
| 273 | + |
| 274 | + // 2. read,sort and write in partition |
| 275 | + List<DataStream<Committable>> dataStreams = new ArrayList<>(); |
| 276 | + |
| 277 | + for (Map.Entry<BinaryRow, DataSplit[]> entry : partitionSplits.entrySet()) { |
| 278 | + DataSplit[] splits = entry.getValue(); |
| 279 | + LinkedHashMap<String, String> partitionSpec = |
| 280 | + partitionComputer.generatePartValues(entry.getKey()); |
| 281 | + // 2.1 generate source for current partition |
| 282 | + Pair<DataStream<RowData>, DataStream<Committable>> sourcePair = |
| 283 | + IncrementalClusterSplitSource.buildSource( |
| 284 | + env, |
| 285 | + table, |
| 286 | + partitionSpec, |
| 287 | + splits, |
| 288 | + options.get(FlinkConnectorOptions.SCAN_PARALLELISM)); |
| 289 | + |
| 290 | + // 2.2 cluster in partition |
| 291 | + Integer sinkParallelism = options.get(FlinkConnectorOptions.SINK_PARALLELISM); |
| 292 | + if (sinkParallelism == null) { |
| 293 | + sinkParallelism = sourcePair.getLeft().getParallelism(); |
| 294 | + } |
| 295 | + TableSortInfo sortInfo = |
| 296 | + new TableSortInfo.Builder() |
| 297 | + .setSortColumns(incrementalClusterManager.clusterKeys()) |
| 298 | + .setSortStrategy(incrementalClusterManager.clusterCurve()) |
| 299 | + .setSinkParallelism(sinkParallelism) |
| 300 | + .setLocalSampleSize(sinkParallelism * localSampleMagnification) |
| 301 | + .setGlobalSampleSize(sinkParallelism * 1000) |
| 302 | + .setRangeNumber(sinkParallelism * 10) |
| 303 | + .build(); |
| 304 | + DataStream<RowData> sorted = |
| 305 | + TableSorter.getSorter(env, sourcePair.getLeft(), table, sortInfo).sort(); |
| 306 | + |
| 307 | + // 2.3 write and then reorganize the committable |
| 308 | + // set parallelism to null, and it'll forward parallelism when doWrite() |
| 309 | + RowAppendTableSink sink = new RowAppendTableSink(table, null, null, null); |
| 310 | + boolean blobAsDescriptor = table.coreOptions().blobAsDescriptor(); |
| 311 | + DataStream<Committable> clusterCommittable = |
| 312 | + sink.doWrite( |
| 313 | + FlinkSinkBuilder.mapToInternalRow( |
| 314 | + sorted, |
| 315 | + table.rowType(), |
| 316 | + blobAsDescriptor, |
| 317 | + table.catalogEnvironment().catalogContext()), |
| 318 | + commitUser, |
| 319 | + null) |
| 320 | + .transform( |
| 321 | + "Rewrite cluster committable", |
| 322 | + new CommittableTypeInfo(), |
| 323 | + new RewriteIncrementalClusterCommittableOperator( |
| 324 | + table, |
| 325 | + compactUnits.entrySet().stream() |
| 326 | + .collect( |
| 327 | + Collectors.toMap( |
| 328 | + Map.Entry::getKey, |
| 329 | + unit -> |
| 330 | + unit.getValue() |
| 331 | + .outputLevel())))); |
| 332 | + dataStreams.add(clusterCommittable); |
| 333 | + dataStreams.add(sourcePair.getRight()); |
| 334 | + } |
| 335 | + |
| 336 | + // 3. commit |
| 337 | + RowAppendTableSink sink = new RowAppendTableSink(table, null, null, null); |
| 338 | + DataStream<Committable> dataStream = dataStreams.get(0); |
| 339 | + for (int i = 1; i < dataStreams.size(); i++) { |
| 340 | + dataStream = dataStream.union(dataStreams.get(i)); |
| 341 | + } |
| 342 | + sink.doCommit(dataStream, commitUser); |
| 343 | + return true; |
| 344 | + } |
| 345 | + |
205 | 346 | protected PartitionPredicate getPartitionPredicate() throws Exception { |
206 | 347 | checkArgument( |
207 | 348 | partitions == null || whereSql == null, |
|
0 commit comments