[HUDI-1147] Modify GenericRecordFullPayloadGenerator to generate vali… (#2045)
* [HUDI-1147] Modify GenericRecordFullPayloadGenerator to generate valid timestamps Co-authored-by: Sivabalan Narayanan <sivabala@uber.com>
This commit is contained in:
committed by
GitHub
parent
da51aa64fc
commit
e33a8f733c
@@ -129,7 +129,7 @@ public class DeltaGenerator implements Serializable {
|
||||
|
||||
public JavaRDD<GenericRecord> generateInserts(Config operation) {
|
||||
int numPartitions = operation.getNumInsertPartitions();
|
||||
long recordsPerPartition = operation.getNumRecordsInsert() / numPartitions;
|
||||
long recordsPerPartition = operation.getNumRecordsInsert();
|
||||
int minPayloadSize = operation.getRecordSize();
|
||||
int startPartition = operation.getStartPartition();
|
||||
|
||||
@@ -140,7 +140,7 @@ public class DeltaGenerator implements Serializable {
|
||||
JavaRDD<GenericRecord> inputBatch = jsc.parallelize(partitionIndexes, numPartitions)
|
||||
.mapPartitionsWithIndex((index, p) -> {
|
||||
return new LazyRecordGeneratorIterator(new FlexibleSchemaRecordGenerationIterator(recordsPerPartition,
|
||||
minPayloadSize, schemaStr, partitionPathFieldNames, (Integer)index));
|
||||
minPayloadSize, schemaStr, partitionPathFieldNames, numPartitions));
|
||||
}, true);
|
||||
|
||||
if (deltaOutputConfig.getInputParallelism() < numPartitions) {
|
||||
|
||||
@@ -20,11 +20,6 @@ package org.apache.hudi.integ.testsuite.generator;
|
||||
|
||||
import org.apache.avro.Schema;
|
||||
import org.apache.avro.generic.GenericRecord;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
import java.util.Iterator;
|
||||
import java.util.List;
|
||||
|
||||
import java.util.HashSet;
|
||||
import java.util.Iterator;
|
||||
@@ -46,17 +41,21 @@ public class FlexibleSchemaRecordGenerationIterator implements Iterator<GenericR
|
||||
private GenericRecord lastRecord;
|
||||
// Partition path field name
|
||||
private Set<String> partitionPathFieldNames;
|
||||
private String firstPartitionPathField;
|
||||
|
||||
public FlexibleSchemaRecordGenerationIterator(long maxEntriesToProduce, String schema) {
|
||||
this(maxEntriesToProduce, GenericRecordFullPayloadGenerator.DEFAULT_PAYLOAD_SIZE, schema, null, 0);
|
||||
}
|
||||
|
||||
public FlexibleSchemaRecordGenerationIterator(long maxEntriesToProduce, int minPayloadSize, String schemaStr,
|
||||
List<String> partitionPathFieldNames, int partitionIndex) {
|
||||
List<String> partitionPathFieldNames, int numPartitions) {
|
||||
this.counter = maxEntriesToProduce;
|
||||
this.partitionPathFieldNames = new HashSet<>(partitionPathFieldNames);
|
||||
if(partitionPathFieldNames != null && partitionPathFieldNames.size() > 0) {
|
||||
this.firstPartitionPathField = partitionPathFieldNames.get(0);
|
||||
}
|
||||
Schema schema = new Schema.Parser().parse(schemaStr);
|
||||
this.generator = new GenericRecordFullPayloadGenerator(schema, minPayloadSize, partitionIndex);
|
||||
this.generator = new GenericRecordFullPayloadGenerator(schema, minPayloadSize, numPartitions);
|
||||
}
|
||||
|
||||
@Override
|
||||
@@ -67,12 +66,18 @@ public class FlexibleSchemaRecordGenerationIterator implements Iterator<GenericR
|
||||
@Override
|
||||
public GenericRecord next() {
|
||||
this.counter--;
|
||||
boolean partitionPathsNonEmpty = partitionPathFieldNames != null && partitionPathFieldNames.size() > 0;
|
||||
if (lastRecord == null) {
|
||||
GenericRecord record = this.generator.getNewPayload(partitionPathFieldNames);
|
||||
GenericRecord record = partitionPathsNonEmpty
|
||||
? this.generator.getNewPayloadWithTimestamp(this.firstPartitionPathField)
|
||||
: this.generator.getNewPayload();
|
||||
lastRecord = record;
|
||||
return record;
|
||||
} else {
|
||||
return this.generator.randomize(lastRecord, this.partitionPathFieldNames);
|
||||
return partitionPathsNonEmpty
|
||||
? this.generator.getUpdatePayloadWithTimestamp(lastRecord,
|
||||
this.partitionPathFieldNames, firstPartitionPathField)
|
||||
: this.generator.getUpdatePayload(lastRecord, this.partitionPathFieldNames);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
@@ -46,9 +46,9 @@ import java.util.concurrent.TimeUnit;
|
||||
*/
|
||||
public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
|
||||
private static Logger LOG = LoggerFactory.getLogger(GenericRecordFullPayloadGenerator.class);
|
||||
|
||||
private static final Logger LOG = LoggerFactory.getLogger(GenericRecordFullPayloadGenerator.class);
|
||||
public static final int DEFAULT_PAYLOAD_SIZE = 1024 * 10; // 10 KB
|
||||
public static final int DEFAULT_NUM_DATE_PARTITIONS = 50;
|
||||
public static final String DEFAULT_HOODIE_IS_DELETED_COL = "_hoodie_is_deleted";
|
||||
protected final Random random = new Random();
|
||||
// The source schema used to generate a payload
|
||||
@@ -58,10 +58,12 @@ public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
// The index of partition for which records are being generated
|
||||
private int partitionIndex = 0;
|
||||
// The size of a full record where every field of a generic record created contains 1 random value
|
||||
private final int estimatedFullPayloadSize;
|
||||
private int estimatedFullPayloadSize;
|
||||
// Number of extra entries to add in a complex/collection field to achieve the desired record size
|
||||
Map<String, Integer> extraEntriesMap = new HashMap<>();
|
||||
|
||||
// The number of unique dates to create
|
||||
private int numDatePartitions = DEFAULT_NUM_DATE_PARTITIONS;
|
||||
// LogicalTypes in Avro 1.8.2
|
||||
private static final String DECIMAL = "decimal";
|
||||
private static final String UUID_NAME = "uuid";
|
||||
@@ -75,6 +77,11 @@ public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
this(schema, DEFAULT_PAYLOAD_SIZE);
|
||||
}
|
||||
|
||||
public GenericRecordFullPayloadGenerator(Schema schema, int minPayloadSize, int numDatePartitions) {
|
||||
this(schema, minPayloadSize);
|
||||
this.numDatePartitions = numDatePartitions;
|
||||
}
|
||||
|
||||
public GenericRecordFullPayloadGenerator(Schema schema, int minPayloadSize) {
|
||||
Pair<Integer, Integer> sizeInfo = new GenericRecordFullPayloadSizeEstimator(schema)
|
||||
.typeEstimateAndNumComplexFields();
|
||||
@@ -83,19 +90,13 @@ public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
if (estimatedFullPayloadSize < minPayloadSize) {
|
||||
int numberOfComplexFields = sizeInfo.getRight();
|
||||
if (numberOfComplexFields < 1) {
|
||||
LOG.warn("The schema does not have any collections/complex fields. "
|
||||
+ "Cannot achieve minPayloadSize => " + minPayloadSize);
|
||||
LOG.warn("The schema does not have any collections/complex fields. Cannot achieve minPayloadSize : {}",
|
||||
minPayloadSize);
|
||||
}
|
||||
|
||||
determineExtraEntriesRequired(numberOfComplexFields, minPayloadSize - estimatedFullPayloadSize);
|
||||
}
|
||||
}
|
||||
|
||||
public GenericRecordFullPayloadGenerator(Schema schema, int minPayloadSize, int partitionIndex) {
|
||||
this(schema, minPayloadSize);
|
||||
this.partitionIndex = partitionIndex;
|
||||
}
|
||||
|
||||
protected static boolean isPrimitive(Schema localSchema) {
|
||||
if (localSchema.getType() != Type.ARRAY
|
||||
&& localSchema.getType() != Type.MAP
|
||||
@@ -131,6 +132,15 @@ public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
return create(baseSchema, partitionPathFieldNames);
|
||||
}
|
||||
|
||||
public GenericRecord getNewPayloadWithTimestamp(String tsFieldName) {
|
||||
return updateTimestamp(create(baseSchema, null), tsFieldName);
|
||||
}
|
||||
|
||||
public GenericRecord getUpdatePayloadWithTimestamp(GenericRecord record, Set<String> blacklistFields,
|
||||
String tsFieldName) {
|
||||
return updateTimestamp(randomize(record, blacklistFields), tsFieldName);
|
||||
}
|
||||
|
||||
protected GenericRecord create(Schema schema, Set<String> partitionPathFieldNames) {
|
||||
GenericRecord result = new GenericData.Record(schema);
|
||||
for (Schema.Field f : schema.getFields()) {
|
||||
@@ -314,6 +324,17 @@ public class GenericRecordFullPayloadGenerator implements Serializable {
|
||||
return genericData.validate(baseSchema, record);
|
||||
}
|
||||
|
||||
/*
|
||||
* Generates a sequential timestamp (daily increment), and updates the timestamp field of the record.
|
||||
* Note: When generating records, number of records to be generated must be more than numDatePartitions * parallelism,
|
||||
* to guarantee that at least numDatePartitions are created.
|
||||
*/
|
||||
public GenericRecord updateTimestamp(GenericRecord record, String fieldName) {
|
||||
long delta = TimeUnit.MILLISECONDS.convert(++partitionIndex % numDatePartitions, TimeUnit.DAYS);
|
||||
record.put(fieldName, System.currentTimeMillis() - delta);
|
||||
return record;
|
||||
}
|
||||
|
||||
/**
|
||||
* Check whether a schema is option. return true if it match the follows: 1. Its type is Type.UNION 2. Has two types 3. Has a NULL type.
|
||||
*/
|
||||
|
||||
Reference in New Issue
Block a user