danny0405 commented on code in PR #8107: URL: https://github.com/apache/hudi/pull/8107#discussion_r1185716426
########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/AutoRecordGenWrapperAvroKeyGenerator.java: ########## @@ -0,0 +1,88 @@ +/* + * 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.hudi.keygen.constant.KeyGeneratorOptions; + +import org.apache.avro.generic.GenericRecord; + +import java.util.List; +import java.util.concurrent.atomic.AtomicBoolean; + +/** + * A wrapper key generator to intercept getRecordKey calls for auto record key generator. + */ +public class AutoRecordGenWrapperAvroKeyGenerator extends BaseKeyGenerator { + + private final BaseKeyGenerator keyGenerator; + private final boolean autoGenerateRecordKeys; + private final AtomicBoolean initializeAutoKeyGenProps = new AtomicBoolean(false); + private int partitionId; + private String instantTime; + private int rowId; + + public AutoRecordGenWrapperAvroKeyGenerator(TypedProperties config, BaseKeyGenerator keyGenerator) { + super(config); + this.keyGenerator = keyGenerator; + this.autoGenerateRecordKeys = !config.containsKey(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()); + } Review Comment: Isn't the attribute `autoGenerateRecordKeys` always true? ########## hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/AutoRecordGenWrapperKeyGenerator.java: ########## @@ -0,0 +1,124 @@ +/* + * 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.hudi.keygen.constant.KeyGeneratorOptions; + +import org.apache.avro.generic.GenericRecord; +import org.apache.spark.sql.Row; +import org.apache.spark.sql.catalyst.InternalRow; +import org.apache.spark.sql.types.StructType; +import org.apache.spark.unsafe.types.UTF8String; + +import java.util.List; +import java.util.concurrent.atomic.AtomicBoolean; + +/** + * A wrapper key generator to intercept getRecordKey calls for auto record key generator. + */ +public class AutoRecordGenWrapperKeyGenerator extends BuiltinKeyGenerator { + + private final BuiltinKeyGenerator builtinKeyGenerator; + private final boolean autoGenerateRecordKeys; + private final AtomicBoolean initializeAutoKeyGenProps = new AtomicBoolean(false); + private int partitionId; + private String instantTime; + private int rowId; + + public AutoRecordGenWrapperKeyGenerator(TypedProperties config, BuiltinKeyGenerator builtinKeyGenerator) { + super(config); + this.builtinKeyGenerator = builtinKeyGenerator; + this.autoGenerateRecordKeys = !config.containsKey(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()); Review Comment: The `autoGenerateRecordKeys` is always true, the factory already ensures that. ########## hudi-client/hudi-spark-client/src/main/scala/org/apache/hudi/HoodieDatasetBulkInsertHelper.scala: ########## @@ -79,11 +83,20 @@ object HoodieDatasetBulkInsertHelper val prependedRdd: RDD[InternalRow] = df.queryExecution.toRdd.mapPartitions { iter => - val keyGenerator = + val sparkKeyGenerator = ReflectionUtils.loadClass(keyGeneratorClassName, new TypedProperties(config.getProps)) - .asInstanceOf[SparkKeyGeneratorInterface] + .asInstanceOf[BuiltinKeyGenerator] + val keyGenerator: BuiltinKeyGenerator = if (autoGenerateRecordKeys) { + val typedProps = new TypedProperties(config.getProps) + typedProps.setProperty(KeyGenUtils.RECORD_KEY_GEN_PARTITION_ID_CONFIG, String.valueOf(TaskContext.getPartitionId())) + typedProps.setProperty(KeyGenUtils.RECORD_KEY_GEN_INSTANT_TIME_CONFIG, instantTime) + new AutoRecordGenWrapperKeyGenerator(typedProps, sparkKeyGenerator).asInstanceOf[BuiltinKeyGenerator] Review Comment: Curious why we didn't have any factory class there? ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/AutoRecordKeyGenerationUtils.scala: ########## @@ -0,0 +1,102 @@ +/* + * 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 +import org.apache.spark.sql.catalyst.InternalRow + +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) { + // de-dup is not supported with auto generation of record keys + if (parameters.getOrElse(HoodieWriteConfig.COMBINE_BEFORE_INSERT.key(), + HoodieWriteConfig.COMBINE_BEFORE_INSERT.defaultValue()).toBoolean) { + throw new HoodieKeyGeneratorException("Enabling " + HoodieWriteConfig.COMBINE_BEFORE_INSERT.key() + " is not supported with auto generation of record keys ") + } + // drop dupes is not supported + if (hoodieConfig.getBoolean(INSERT_DROP_DUPS)) { + throw new HoodieKeyGeneratorException("Enabling " + INSERT_DROP_DUPS.key() + " is not supported with auto generation of record keys ") + } + // virtual keys are not supported with auto generation of record keys. + if (!parameters.getOrElse(HoodieTableConfig.POPULATE_META_FIELDS.key(), HoodieTableConfig.POPULATE_META_FIELDS.defaultValue().toString).toBoolean) { + throw new HoodieKeyGeneratorException("Disabling " + HoodieTableConfig.POPULATE_META_FIELDS.key() + " is not supported with auto generation of record keys") + } + } + } + + /** + * Auto Generate record keys when auto generation config is enabled. + * <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. + * + * @param autoGenerateKeys true if auto generation of record keys is enabled. false otherwise. + * @param genRecsItr Iterator of GenericRecords. + * @param instantTime commit time of the batch. + * @param sparkPartitionId spark partition Id of interest. + * @return Iterator of Pair of GenericRecord and Optionally generated record key. + */ + def mayBeAutoGenerateRecordKeys(autoGenerateKeys : Boolean, genRecsItr: Iterator[GenericRecord], instantTime: String, + sparkPartitionId: Integer): Iterator[(GenericRecord, Option[String])] = { Review Comment: Can be removed. ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/DataSourceOptions.scala: ########## @@ -844,8 +844,8 @@ object DataSourceOptionsHelper { */ def fetchMissingWriteConfigsFromTableConfig(tableConfig: HoodieTableConfig, params: Map[String, String]) : Map[String, String] = { val missingWriteConfigs = scala.collection.mutable.Map[String, String]() - if (!params.contains(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()) && tableConfig.getRecordKeyFieldProp != null) { - missingWriteConfigs ++= Map(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key() -> tableConfig.getRecordKeyFieldProp) + if (!params.contains(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key()) && tableConfig.getRawRecordKeyFieldProp != null) { + missingWriteConfigs ++= Map(KeyGeneratorOptions.RECORDKEY_FIELD_NAME.key() -> tableConfig.getRawRecordKeyFieldProp) Review Comment: We need to dig out why the `_hoodie_record_key` is used as a backend field name before, there it actually can take effect? ########## hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/TestDataSourceDefaults.scala: ########## @@ -259,17 +249,6 @@ class TestDataSourceDefaults extends ScalaAssertionSupport { keyGen.getRecordKey(internalRow, structType) } - // Record's key field not specified - assertThrows(classOf[IllegalArgumentException]) { - val props = new TypedProperties() - props.setProperty(DataSourceWriteOptions.PARTITIONPATH_FIELD.key, "partitionField") - val keyGen = new ComplexKeyGenerator(props) - - keyGen.getKey(baseRecord) Review Comment: add it back to make the test pass. ########## hudi-spark-datasource/hudi-spark/src/test/java/org/apache/hudi/keygen/TestComplexKeyGenerator.java: ########## @@ -75,11 +75,6 @@ public void testNullPartitionPathFields() { Assertions.assertThrows(IllegalArgumentException.class, () -> new ComplexKeyGenerator(getPropertiesWithoutPartitionPathProp())); } - @Test - public void testNullRecordKeyFields() { - Assertions.assertThrows(IllegalArgumentException.class, () -> new ComplexKeyGenerator(getPropertiesWithoutRecordKeyProp())); - } Review Comment: Have not addressed? ########## hudi-client/hudi-spark-client/src/main/java/org/apache/hudi/keygen/SimpleKeyGenerator.java: ########## @@ -109,17 +110,17 @@ public UTF8String getPartitionPath(InternalRow row, StructType schema) { return combinePartitionPathUnsafe(rowAccessor.getRecordPartitionPathValues(row)); } - private static void validatePartitionPath(String partitionPathField) { + private void validatePartitionPath(String partitionPathField) { Review Comment: It is a good convention to alway make utilities `static`! ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/AutoRecordKeyGenerationUtils.scala: ########## @@ -0,0 +1,102 @@ +/* + * 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 +import org.apache.spark.sql.catalyst.InternalRow + +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) { + // de-dup is not supported with auto generation of record keys + if (parameters.getOrElse(HoodieWriteConfig.COMBINE_BEFORE_INSERT.key(), + HoodieWriteConfig.COMBINE_BEFORE_INSERT.defaultValue()).toBoolean) { + throw new HoodieKeyGeneratorException("Enabling " + HoodieWriteConfig.COMBINE_BEFORE_INSERT.key() + " is not supported with auto generation of record keys ") + } + // drop dupes is not supported + if (hoodieConfig.getBoolean(INSERT_DROP_DUPS)) { + throw new HoodieKeyGeneratorException("Enabling " + INSERT_DROP_DUPS.key() + " is not supported with auto generation of record keys ") + } + // virtual keys are not supported with auto generation of record keys. + if (!parameters.getOrElse(HoodieTableConfig.POPULATE_META_FIELDS.key(), HoodieTableConfig.POPULATE_META_FIELDS.defaultValue().toString).toBoolean) { + throw new HoodieKeyGeneratorException("Disabling " + HoodieTableConfig.POPULATE_META_FIELDS.key() + " is not supported with auto generation of record keys") + } + } + } + + /** + * Auto Generate record keys when auto generation config is enabled. + * <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. + * + * @param autoGenerateKeys true if auto generation of record keys is enabled. false otherwise. + * @param genRecsItr Iterator of GenericRecords. + * @param instantTime commit time of the batch. + * @param sparkPartitionId spark partition Id of interest. + * @return Iterator of Pair of GenericRecord and Optionally generated record key. + */ + def mayBeAutoGenerateRecordKeys(autoGenerateKeys : Boolean, genRecsItr: Iterator[GenericRecord], instantTime: String, + sparkPartitionId: Integer): Iterator[(GenericRecord, Option[String])] = { + + genRecsItr.map(avroRecord => { + (avroRecord, Option.empty) + }) + } + + def mayBeAutoGenerateRecordKeysForSparkRow(autoGenerateKeys : Boolean, internalRowsItr: Iterator[InternalRow], instantTime: String, + sparkPartitionId: Integer): Iterator[(InternalRow, Option[String])] = { Review Comment: Can be removed. ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/HoodieSparkSqlWriter.scala: ########## @@ -1118,47 +1124,74 @@ object HoodieSparkSqlWriter { Some(writerSchema)) avroRecords.mapPartitions(it => { + val sparkPartitionId = TaskContext.getPartitionId() + val keyGenProps = new TypedProperties(config.getProps) + if (autoGenerateRecordKeys) { + keyGenProps.setProperty(KeyGenUtils.RECORD_KEY_GEN_PARTITION_ID_CONFIG, String.valueOf(sparkPartitionId)) Review Comment: Guess the initialization of `keyGenProps` can be moved out of the `mapPartition` call? ########## hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/TestDataSourceDefaults.scala: ########## @@ -89,16 +89,6 @@ class TestDataSourceDefaults extends ScalaAssertionSupport { } } - { - // Record's key field not specified - val props = new TypedProperties() - props.setProperty(DataSourceWriteOptions.PARTITIONPATH_FIELD.key(), "partitionField") - - assertThrows(classOf[IllegalArgumentException]) { - new SimpleKeyGenerator(props) Review Comment: add it back to make the test pass. ########## hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/HoodieSparkSqlWriter.scala: ########## @@ -1118,47 +1124,74 @@ object HoodieSparkSqlWriter { Some(writerSchema)) avroRecords.mapPartitions(it => { + val sparkPartitionId = TaskContext.getPartitionId() + val keyGenProps = new TypedProperties(config.getProps) + if (autoGenerateRecordKeys) { + keyGenProps.setProperty(KeyGenUtils.RECORD_KEY_GEN_PARTITION_ID_CONFIG, String.valueOf(sparkPartitionId)) + keyGenProps.setProperty(KeyGenUtils.RECORD_KEY_GEN_INSTANT_TIME_CONFIG, instantTime) + } + val keyGenerator = HoodieSparkKeyGeneratorFactory.createKeyGenerator(keyGenProps) + .asInstanceOf[BaseKeyGenerator] + val dataFileSchema = new Schema.Parser().parse(dataFileSchemaStr) val consistentLogicalTimestampEnabled = parameters.getOrElse( DataSourceWriteOptions.KEYGENERATOR_CONSISTENT_LOGICAL_TIMESTAMP_ENABLED.key(), DataSourceWriteOptions.KEYGENERATOR_CONSISTENT_LOGICAL_TIMESTAMP_ENABLED.defaultValue()).toBoolean - it.map { avroRecord => + val recordRecordKeyPairItr = it.map(avroRecord => { + (avroRecord, keyGenerator.getRecordKey(avroRecord)) + }) Review Comment: Why we eagerly materialize the record key first? ########## hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/spark/sql/hudi/TestHoodieOptionConfig.scala: ########## @@ -109,16 +109,6 @@ class TestHoodieOptionConfig extends SparkClientFunctionalTestHarness { StructField("dt", StringType, true)) ) - // miss primaryKey parameter - val sqlOptions1 = baseSqlOptions ++ Map( - "type" -> "mor" - ) - - val e1 = intercept[IllegalArgumentException] { - HoodieOptionConfig.validateTable(spark, schema, sqlOptions1) Review Comment: add it back to make the test pass. ########## hudi-client/hudi-client-common/src/main/java/org/apache/hudi/keygen/KeyGenUtils.java: ########## @@ -61,10 +64,27 @@ public class KeyGenUtils { */ public static KeyGeneratorType inferKeyGeneratorType( String recordsKeyFields, String partitionFields) { + boolean autoGenerateRecordKeys = recordsKeyFields == null; + if (autoGenerateRecordKeys) { Review Comment: Make `recordsKeyFields` param optional ? ########## hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/spark/sql/hudi/TestInsertTable.scala: ########## @@ -1267,4 +1268,41 @@ class TestInsertTable extends HoodieSparkSqlTestBase { }) } } + + test("Test Insert Into with auto generate record keys") { + withTempDir { tmp => + val tableName = generateTableName + // Create a partitioned table + spark.sql( + s""" + |create table $tableName ( + | id int, + | dt string, + | name string, + | price double, + | ts long + |) using hudi + | partitioned by (dt) + | location '${tmp.getCanonicalPath}' + """.stripMargin) Review Comment: test all cases including: simple keygen/complex key gen and non-partitioned key gen ########## hudi-utilities/src/main/java/org/apache/hudi/utilities/deltastreamer/DeltaSync.java: ########## @@ -867,18 +894,20 @@ private Pair<Option<String>, JavaRDD<WriteStatus>> writeToSink(JavaRDD<HoodieRec * * @return Instant time of the commit */ - private String startCommit() { + private String startCommit(String instantTime, boolean retryEnabled) { Review Comment: Is this change related with the PR? ########## 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: Can we move the check `if (autoGenerateRecordKeys) {` out of the method? ########## hudi-spark-datasource/hudi-spark/src/test/scala/org/apache/hudi/TestDataSourceDefaults.scala: ########## @@ -487,16 +466,6 @@ class TestDataSourceDefaults extends ScalaAssertionSupport { assertEquals(UTF8String.fromString(expectedKey.getPartitionPath), keyGen.getPartitionPath(internalRow, structType)) } - { - // Record's key field not specified - val props = new TypedProperties() - props.setProperty(DataSourceWriteOptions.PARTITIONPATH_FIELD.key, "partitionField") Review Comment: add it back to make the test pass. -- 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]
