codope commented on code in PR #8107:
URL: https://github.com/apache/hudi/pull/8107#discussion_r1143727808


##########
hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/HoodieSparkSqlWriter.scala:
##########
@@ -1116,31 +1124,47 @@ object HoodieSparkSqlWriter {
           Some(writerSchema))
 
         avroRecords.mapPartitions(it => {
+          val sparkPartitionId = TaskContext.getPartitionId()
+
           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 =>
+          // generate record keys is auto generation is enabled.

Review Comment:
   ```suggestion
             // generate record keys if auto generation is enabled.
   ```



##########
hudi-spark-datasource/hudi-spark-common/src/main/scala/org/apache/hudi/AutoRecordKeyGenerationUtils.scala:
##########
@@ -0,0 +1,103 @@
+/*
+ * 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.HoodieException
+import org.apache.spark.TaskContext
+
+object AutoRecordKeyGenerationUtils {
+
+   // supported operation types when auto generation of record keys is enabled.
+   val supportedOperations: Set[String] =
+    Set(WriteOperationType.INSERT, WriteOperationType.BULK_INSERT, 
WriteOperationType.DELETE,
+      WriteOperationType.INSERT_OVERWRITE, 
WriteOperationType.INSERT_OVERWRITE_TABLE,
+      WriteOperationType.DELETE_PARTITION).map(_.name())
+
+  def validateParamsForAutoGenerationOfRecordKeys(parameters: Map[String, 
String],
+                                                  operation: 
WriteOperationType, hoodieConfig: HoodieConfig): Unit = {
+    val autoGenerateRecordKeys: Boolean = 
parameters.getOrElse(HoodieTableConfig.AUTO_GENERATE_RECORD_KEYS.key(),
+      HoodieTableConfig.AUTO_GENERATE_RECORD_KEYS.defaultValue()).toBoolean
+
+    if (autoGenerateRecordKeys) {
+      // check for supported operations.
+      if (!supportedOperations.contains(operation.name())) {
+        throw new HoodieException(operation.name() + " is not supported with 
Auto generation of record keys. "
+          + "Supported operations are : " + supportedOperations)
+      }
+      // 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 HoodieException("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 HoodieException("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 HoodieException("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.
+   * @return Iterator of Pair of GenericRecord and Optionally generated record 
key.
+   */
+  def mayBeAutoGenerateRecordKeys(autoGenerateKeys : Boolean, genRecsItr: 
Iterator[GenericRecord], instantTime: String): Iterator[(GenericRecord, 
Option[String])] = {
+    var rowId = 0
+    val sparkPartitionId = TaskContext.getPartitionId()
+
+    // we will override record keys if auto generation if keys is enabled.
+    genRecsItr.map(avroRecord =>
+      if (autoGenerateKeys) {
+        val recordKey : String = HoodieRecord.generateSequenceId(instantTime, 
sparkPartitionId, rowId)
+        rowId += 1

Review Comment:
   We should check why it is Atomic in some of the write handles, eg. 
HoodieAppendHandle. If it's not needed then we can do away with this overhead.



##########
hudi-common/src/main/java/org/apache/hudi/common/table/HoodieTableConfig.java:
##########
@@ -260,6 +260,18 @@ public class HoodieTableConfig extends HoodieConfig {
       .sinceVersion("0.13.0")
       .withDocumentation("The metadata of secondary indexes");
 
+  public static final ConfigProperty<String> AUTO_GENERATE_RECORD_KEYS = 
ConfigProperty
+      .key("hoodie.table.auto.generate.record.keys")
+      .defaultValue("false")
+      .withDocumentation("Enables automatic generation of the record-keys in 
cases when dataset bears "

Review Comment:
   Got it. Somehow, feels counter-intuitive that I enabled auto record keys and 
then also need to set keygen. But, I guess we can't do much here. We should 
think of a way to infer timestamp/custom keygen class based on how user set 
partitionpath field or other configs in `KeyGeneratorOptions.Config`.



-- 
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]

Reply via email to