This is an automated email from the ASF dual-hosted git repository. forwardxu pushed a commit to branch release-0.12.1 in repository https://gitbox.apache.org/repos/asf/hudi.git
commit f02fef936b7db27137229c4bd64397a7456b915c Author: superche <super...@tencent.com> AuthorDate: Thu Nov 17 16:02:31 2022 +0800 fix none index partition format --- .../java/org/apache/hudi/keygen/EmptyAvroKeyGenerator.java | 11 ++++++++--- .../apache/hudi/keygen/TimestampBasedAvroKeyGenerator.java | 4 ++-- .../main/java/org/apache/hudi/keygen/EmptyKeyGenerator.java | 3 ++- .../main/java/org/apache/hudi/table/HoodieTableFactory.java | 7 +++++++ 4 files changed, 19 insertions(+), 6 deletions(-) diff --git a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/EmptyAvroKeyGenerator.java b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/EmptyAvroKeyGenerator.java index 01536f95e4..6759c3dc8e 100644 --- a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/EmptyAvroKeyGenerator.java +++ b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/EmptyAvroKeyGenerator.java @@ -18,6 +18,7 @@ package org.apache.hudi.keygen; +import java.io.IOException; import org.apache.avro.generic.GenericRecord; import org.apache.hudi.common.model.HoodieKey; @@ -36,13 +37,13 @@ import java.util.stream.Collectors; /** * Avro key generator for empty record key Hudi tables. */ -public class EmptyAvroKeyGenerator extends BaseKeyGenerator { +public class EmptyAvroKeyGenerator extends TimestampBasedAvroKeyGenerator { private static final Logger LOG = LogManager.getLogger(EmptyAvroKeyGenerator.class); public static final String EMPTY_RECORD_KEY = HoodieKey.EMPTY_RECORD_KEY; private static final List<String> EMPTY_RECORD_KEY_FIELD_LIST = Collections.emptyList(); - public EmptyAvroKeyGenerator(TypedProperties props) { + public EmptyAvroKeyGenerator(TypedProperties props) throws IOException { super(props); if (config.containsKey(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key())) { LOG.warn(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key() + " will be ignored while using " @@ -60,6 +61,10 @@ public class EmptyAvroKeyGenerator extends BaseKeyGenerator { @Override public String getPartitionPath(GenericRecord record) { - return KeyGenUtils.getRecordPartitionPath(record, getPartitionPathFields(), hiveStylePartitioning, encodePartitionPath, isConsistentLogicalTimestampEnabled()); + if (this.timestampType == TimestampType.NO_TIMESTAMP) { + return KeyGenUtils.getRecordPartitionPath(record, getPartitionPathFields(), hiveStylePartitioning, encodePartitionPath, isConsistentLogicalTimestampEnabled()); + } else { + return super.getPartitionPath(record); + } } } diff --git a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/TimestampBasedAvroKeyGenerator.java b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/TimestampBasedAvroKeyGenerator.java index 60ccc694f9..77863fd869 100644 --- a/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/TimestampBasedAvroKeyGenerator.java +++ b/hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/TimestampBasedAvroKeyGenerator.java @@ -49,11 +49,11 @@ import static java.util.concurrent.TimeUnit.SECONDS; */ public class TimestampBasedAvroKeyGenerator extends SimpleAvroKeyGenerator { public enum TimestampType implements Serializable { - UNIX_TIMESTAMP, DATE_STRING, MIXED, EPOCHMILLISECONDS, SCALAR + UNIX_TIMESTAMP, DATE_STRING, MIXED, EPOCHMILLISECONDS, SCALAR, NO_TIMESTAMP } private final TimeUnit timeUnit; - private final TimestampType timestampType; + protected final TimestampType timestampType; private final String outputDateFormat; private transient Option<DateTimeFormatter> inputFormatter; private transient DateTimeFormatter partitionFormatter; diff --git a/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/EmptyKeyGenerator.java b/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/EmptyKeyGenerator.java index 9e4090a537..2ba0d5cf32 100644 --- a/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/EmptyKeyGenerator.java +++ b/hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/EmptyKeyGenerator.java @@ -18,6 +18,7 @@ package org.apache.hudi.keygen; +import java.io.IOException; import org.apache.avro.generic.GenericRecord; import org.apache.hudi.common.config.TypedProperties; import org.apache.hudi.keygen.constant.KeyGeneratorOptions; @@ -38,7 +39,7 @@ public class EmptyKeyGenerator extends BuiltinKeyGenerator { private final EmptyAvroKeyGenerator emptyAvroKeyGenerator; - public EmptyKeyGenerator(TypedProperties config) { + public EmptyKeyGenerator(TypedProperties config) throws IOException { super(config); this.emptyAvroKeyGenerator = new EmptyAvroKeyGenerator(config); this.recordKeyFields = emptyAvroKeyGenerator.getRecordKeyFieldNames(); diff --git a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableFactory.java b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableFactory.java index b0380c5878..612aa623e5 100644 --- a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableFactory.java +++ b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableFactory.java @@ -196,6 +196,13 @@ public class HoodieTableFactory implements DynamicTableSourceFactory, DynamicTab conf.setString(FlinkOptions.KEYGEN_CLASS_NAME, EmptyAvroKeyGenerator.class.getName()); conf.setString(FlinkOptions.OPERATION, WriteOperationType.INSERT.value()); conf.setString(FlinkOptions.MERGE_TYPE, REALTIME_SKIP_MERGE); + TimestampBasedAvroKeyGenerator.TimestampType timestampType = TimestampBasedAvroKeyGenerator.TimestampType + .valueOf(conf.toMap().getOrDefault(KeyGeneratorOptions.Config.TIMESTAMP_TYPE_FIELD_PROP, TimestampBasedAvroKeyGenerator.TimestampType.NO_TIMESTAMP.name())); + if (timestampType == TimestampBasedAvroKeyGenerator.TimestampType.NO_TIMESTAMP) { + conf.setString(KeyGeneratorOptions.Config.TIMESTAMP_TYPE_FIELD_PROP, TimestampBasedAvroKeyGenerator.TimestampType.NO_TIMESTAMP.name()); + // the option is actually useless, it only works for validation + conf.setString(KeyGeneratorOptions.Config.TIMESTAMP_OUTPUT_DATE_FORMAT_PROP, FlinkOptions.PARTITION_FORMAT_HOUR); + } } }