[HUDI-4372] Enable matadata table by default for flink (#6066)
This commit is contained in:
@@ -103,8 +103,8 @@ public class FlinkOptions extends HoodieConfig {
|
||||
public static final ConfigOption<Boolean> METADATA_ENABLED = ConfigOptions
|
||||
.key("metadata.enabled")
|
||||
.booleanType()
|
||||
.defaultValue(false)
|
||||
.withDescription("Enable the internal metadata table which serves table metadata like level file listings, default false");
|
||||
.defaultValue(true)
|
||||
.withDescription("Enable the internal metadata table which serves table metadata like level file listings, default enabled");
|
||||
|
||||
public static final ConfigOption<Integer> METADATA_COMPACTION_DELTA_COMMITS = ConfigOptions
|
||||
.key("metadata.compaction.delta_commits")
|
||||
|
||||
@@ -51,6 +51,7 @@ import org.jetbrains.annotations.Nullable;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.Serializable;
|
||||
import java.util.Arrays;
|
||||
import java.util.Collection;
|
||||
@@ -181,8 +182,10 @@ public class StreamWriteOperatorCoordinator
|
||||
this.gateways = new SubtaskGateway[this.parallelism];
|
||||
// init table, create if not exists.
|
||||
this.metaClient = initTableIfNotExists(this.conf);
|
||||
this.ckpMetadata = initCkpMetadata(this.metaClient);
|
||||
// the write client must create after the table creation
|
||||
this.writeClient = StreamerUtil.createWriteClient(conf);
|
||||
initMetadataTable(this.writeClient);
|
||||
this.tableState = TableState.create(conf);
|
||||
// start the executor
|
||||
this.executor = NonThrownExecutor.builder(LOG)
|
||||
@@ -192,11 +195,6 @@ public class StreamWriteOperatorCoordinator
|
||||
if (tableState.syncHive) {
|
||||
initHiveSync();
|
||||
}
|
||||
if (tableState.syncMetadata) {
|
||||
initMetadataSync();
|
||||
}
|
||||
this.ckpMetadata = CkpMetadata.getInstance(this.metaClient.getFs(), metaClient.getBasePath());
|
||||
this.ckpMetadata.bootstrap(this.metaClient);
|
||||
}
|
||||
|
||||
@Override
|
||||
@@ -352,8 +350,14 @@ public class StreamWriteOperatorCoordinator
|
||||
hiveSyncContext.hiveSyncTool().syncHoodieTable();
|
||||
}
|
||||
|
||||
private void initMetadataSync() {
|
||||
this.writeClient.initMetadataWriter();
|
||||
private static void initMetadataTable(HoodieFlinkWriteClient<?> writeClient) {
|
||||
writeClient.initMetadataTable();
|
||||
}
|
||||
|
||||
private static CkpMetadata initCkpMetadata(HoodieTableMetaClient metaClient) throws IOException {
|
||||
CkpMetadata ckpMetadata = CkpMetadata.getInstance(metaClient.getFs(), metaClient.getBasePath());
|
||||
ckpMetadata.bootstrap(metaClient);
|
||||
return ckpMetadata;
|
||||
}
|
||||
|
||||
private void reset() {
|
||||
|
||||
@@ -87,8 +87,8 @@ public class ExpressionEvaluator {
|
||||
* 2. bind the field reference;
|
||||
* 3. bind the column stats.
|
||||
*
|
||||
* <p>Normalize the expression to simplify the following decision logic:
|
||||
* always put the literal expression in the right.
|
||||
* <p>Normalize the expression to simplify the subsequent decision logic:
|
||||
* always put the literal expression in the RHS.
|
||||
*/
|
||||
public static Evaluator bindCall(CallExpression call, RowData indexRow, RowType.RowField[] queryFields) {
|
||||
FunctionDefinition funDef = call.getFunctionDefinition();
|
||||
|
||||
@@ -102,6 +102,7 @@ public class ITTestHoodieFlinkCompactor {
|
||||
tableEnv.getConfig().getConfiguration()
|
||||
.setInteger(ExecutionConfigOptions.TABLE_EXEC_RESOURCE_DEFAULT_PARALLELISM, 1);
|
||||
Map<String, String> options = new HashMap<>();
|
||||
options.put(FlinkOptions.COMPACTION_SCHEDULE_ENABLED.key(), "false");
|
||||
options.put(FlinkOptions.COMPACTION_ASYNC_ENABLED.key(), "false");
|
||||
options.put(FlinkOptions.PATH.key(), tempFile.getAbsolutePath());
|
||||
options.put(FlinkOptions.TABLE_TYPE.key(), "MERGE_ON_READ");
|
||||
@@ -214,6 +215,7 @@ public class ITTestHoodieFlinkCompactor {
|
||||
tableEnv.getConfig().getConfiguration()
|
||||
.setInteger(ExecutionConfigOptions.TABLE_EXEC_RESOURCE_DEFAULT_PARALLELISM, 1);
|
||||
Map<String, String> options = new HashMap<>();
|
||||
options.put(FlinkOptions.COMPACTION_SCHEDULE_ENABLED.key(), "false");
|
||||
options.put(FlinkOptions.COMPACTION_ASYNC_ENABLED.key(), "false");
|
||||
options.put(FlinkOptions.PATH.key(), tempFile.getAbsolutePath());
|
||||
options.put(FlinkOptions.TABLE_TYPE.key(), "MERGE_ON_READ");
|
||||
@@ -222,6 +224,7 @@ public class ITTestHoodieFlinkCompactor {
|
||||
tableEnv.executeSql(hoodieTableDDL);
|
||||
tableEnv.executeSql(TestSQL.INSERT_T1).await();
|
||||
|
||||
// wait for the asynchronous commit to finish
|
||||
TimeUnit.SECONDS.sleep(3);
|
||||
|
||||
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
|
||||
@@ -247,6 +250,7 @@ public class ITTestHoodieFlinkCompactor {
|
||||
+ "('id13','Jenny',72,TIMESTAMP '1970-01-01 00:00:10','par5')";
|
||||
tableEnv.executeSql(insertT1ForNewPartition).await();
|
||||
|
||||
// wait for the asynchronous commit to finish
|
||||
TimeUnit.SECONDS.sleep(3);
|
||||
|
||||
compactionInstantTimeList.add(scheduleCompactionPlan(metaClient, writeClient));
|
||||
|
||||
@@ -480,6 +480,8 @@ public class TestInputFormat {
|
||||
options.put(FlinkOptions.ARCHIVE_MIN_COMMITS.key(), "3");
|
||||
options.put(FlinkOptions.ARCHIVE_MAX_COMMITS.key(), "4");
|
||||
options.put(FlinkOptions.CLEAN_RETAIN_COMMITS.key(), "2");
|
||||
// disable the metadata table to make the archiving behavior deterministic
|
||||
options.put(FlinkOptions.METADATA_ENABLED.key(), "false");
|
||||
options.put("hoodie.commits.archival.batch", "1");
|
||||
beforeEach(HoodieTableType.COPY_ON_WRITE, options);
|
||||
|
||||
|
||||
@@ -29,6 +29,7 @@ import org.apache.hudi.common.table.timeline.TimelineMetadataUtils;
|
||||
import org.apache.hudi.common.util.Option;
|
||||
import org.apache.hudi.configuration.FlinkOptions;
|
||||
import org.apache.hudi.exception.HoodieIOException;
|
||||
import org.apache.hudi.metadata.FlinkHoodieBackedTableMetadataWriter;
|
||||
import org.apache.hudi.table.HoodieFlinkTable;
|
||||
import org.apache.hudi.util.CompactionUtil;
|
||||
import org.apache.hudi.util.FlinkTables;
|
||||
@@ -77,6 +78,11 @@ public class TestCompactionUtil {
|
||||
|
||||
this.table = FlinkTables.createTable(conf);
|
||||
this.metaClient = table.getMetaClient();
|
||||
// initialize the metadata table path
|
||||
if (conf.getBoolean(FlinkOptions.METADATA_ENABLED)) {
|
||||
FlinkHoodieBackedTableMetadataWriter.create(table.getHadoopConf(), table.getConfig(),
|
||||
table.getContext(), Option.empty(), Option.empty());
|
||||
}
|
||||
}
|
||||
|
||||
@Test
|
||||
|
||||
Reference in New Issue
Block a user