vinothchandar commented on code in PR #8107: URL: https://github.com/apache/hudi/pull/8107#discussion_r1187547491
########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/AutoRecordGenWrapperAvroKeyGenerator.java: ########## @@ -0,0 +1,83 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you 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 org.apache.hudi.keygen; + +import org.apache.hudi.common.config.TypedProperties; +import org.apache.hudi.common.model.HoodieRecord; + +import org.apache.avro.generic.GenericRecord; + +import java.util.List; + +/** + * A wrapper key generator to intercept getRecordKey calls for auto record key generator. + * <ol> + * <li>Generated keys will be unique not only w/in provided [[org.apache.spark.sql.DataFrame]], but + * globally unique w/in the target table</li> + * <li>Generated keys have minimal overhead (to compute, persist and read)</li> + * </ol> + * + * Keys adhere to the following format: + * + * [instantTime]_[PartitionId]_[RowId] + * + * where + * instantTime refers to the commit time of the batch being ingested. + * PartitionId refers to spark's partition Id. + * RowId refers to the row index within the spark partition. + */ +public class AutoRecordGenWrapperAvroKeyGenerator extends BaseKeyGenerator { Review Comment: rename `WrappingKeyGenerator` to make it easier to read? its best to decouple names for broad implementations like these from specific features for which they are added. ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/CustomAvroKeyGenerator.java: ########## @@ -55,7 +57,11 @@ public enum PartitionKeyType { public CustomAvroKeyGenerator(TypedProperties props) { super(props); - this.recordKeyFields = Arrays.stream(props.getString(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()).split(",")).map(String::trim).collect(Collectors.toList()); + this.recordKeyFields = Option.ofNullable(props.getString(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key(), null)) Review Comment: please move this logic into an util method and not repeat 5 lines in each file? ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/factory/HoodieAvroKeyGeneratorFactory.java: ########## @@ -71,22 +73,36 @@ public static KeyGenerator createAvroKeyGeneratorByType(TypedProperties props) t throw new HoodieKeyGeneratorException("Unsupported keyGenerator Type " + keyGeneratorType); } + BaseKeyGenerator keyGenerator = null; + switch (keyGeneratorTypeEnum) { case SIMPLE: - return new SimpleAvroKeyGenerator(props); + keyGenerator = new SimpleAvroKeyGenerator(props); + break; case COMPLEX: - return new ComplexAvroKeyGenerator(props); + keyGenerator = new ComplexAvroKeyGenerator(props); + break; case TIMESTAMP: - return new TimestampBasedAvroKeyGenerator(props); + keyGenerator = new TimestampBasedAvroKeyGenerator(props); + break; case CUSTOM: - return new CustomAvroKeyGenerator(props); + keyGenerator = new CustomAvroKeyGenerator(props); + break; case NON_PARTITION: - return new NonpartitionedAvroKeyGenerator(props); + keyGenerator = new NonpartitionedAvroKeyGenerator(props); + break; case GLOBAL_DELETE: - return new GlobalAvroDeleteKeyGenerator(props); + keyGenerator = new GlobalAvroDeleteKeyGenerator(props); + break; default: throw new HoodieKeyGeneratorException("Unsupported keyGenerator Type " + keyGeneratorType); } + + if (KeyGenUtils.enableAutoGenerateRecordKeys(props)) { Review Comment: should we always use the wrapper key generator and move all this logic into that class? ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/AutoRecordKeyGenerationUtils.scala: ########## @@ -0,0 +1,93 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you 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 org.apache.hudi + +import org.apache.avro.generic.GenericRecord +import org.apache.hudi.DataSourceWriteOptions.INSERT_DROP_DUPS +import org.apache.hudi.common.config.HoodieConfig +import org.apache.hudi.common.model.{HoodieRecord, WriteOperationType} +import org.apache.hudi.common.table.HoodieTableConfig +import org.apache.hudi.config.HoodieWriteConfig +import org.apache.hudi.exception.HoodieKeyGeneratorException +import org.apache.hudi.keygen.constant.KeyGeneratorOptions + +object AutoRecordKeyGenerationUtils { + + def validateParamsForAutoGenerationOfRecordKeys(parameters: Map[String, String], hoodieConfig: HoodieConfig): Unit = { + val autoGenerateRecordKeys = !parameters.contains(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()) // if record key is not configured, + // hudi will auto generate. + + if (autoGenerateRecordKeys) { Review Comment: This needs resolution? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: [email protected] For queries about this service, please contact Infrastructure at: [email protected]
