-
Notifications
You must be signed in to change notification settings - Fork 4.3k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Add Iceberg support for name-based mapping schema #33315
Open
regadas
wants to merge
1
commit into
apache:master
Choose a base branch
from
regadas:fix_ice_namemapping
base: master
Could not load branches
Branch not found: {{ refName }}
Loading
Could not load tags
Nothing to show
Loading
Are you sure you want to change the base?
Some commits from the old base branch may be removed from the timeline,
and old review comments may become outdated.
+182
−9
Open
Changes from all commits
Commits
File filter
Filter by extension
Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -20,11 +20,15 @@ | |
import static org.hamcrest.MatcherAssert.assertThat; | ||
import static org.hamcrest.Matchers.containsInAnyOrder; | ||
|
||
import java.io.File; | ||
import java.io.IOException; | ||
import java.util.List; | ||
import java.util.Map; | ||
import java.util.UUID; | ||
import java.util.stream.Collectors; | ||
import java.util.stream.Stream; | ||
import org.apache.avro.generic.GenericData; | ||
import org.apache.avro.generic.GenericRecord; | ||
import org.apache.beam.sdk.coders.RowCoder; | ||
import org.apache.beam.sdk.schemas.Schema; | ||
import org.apache.beam.sdk.testing.PAssert; | ||
|
@@ -33,10 +37,28 @@ | |
import org.apache.beam.sdk.transforms.ParDo; | ||
import org.apache.beam.sdk.values.PCollection; | ||
import org.apache.beam.sdk.values.Row; | ||
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.collect.ImmutableList; | ||
import org.apache.beam.vendor.guava.v32_1_2_jre.com.google.common.collect.ImmutableMap; | ||
import org.apache.hadoop.conf.Configuration; | ||
import org.apache.hadoop.fs.Path; | ||
import org.apache.iceberg.CatalogUtil; | ||
import org.apache.iceberg.DataFile; | ||
import org.apache.iceberg.DataFiles; | ||
import org.apache.iceberg.FileFormat; | ||
import org.apache.iceberg.Metrics; | ||
import org.apache.iceberg.MetricsConfig; | ||
import org.apache.iceberg.PartitionSpec; | ||
import org.apache.iceberg.Table; | ||
import org.apache.iceberg.TableProperties; | ||
import org.apache.iceberg.catalog.TableIdentifier; | ||
import org.apache.iceberg.data.Record; | ||
import org.apache.iceberg.hadoop.HadoopInputFile; | ||
import org.apache.iceberg.mapping.MappedField; | ||
import org.apache.iceberg.mapping.NameMapping; | ||
import org.apache.iceberg.mapping.NameMappingParser; | ||
import org.apache.iceberg.parquet.ParquetUtil; | ||
import org.apache.parquet.avro.AvroParquetWriter; | ||
import org.apache.parquet.hadoop.ParquetWriter; | ||
import org.junit.ClassRule; | ||
import org.junit.Rule; | ||
import org.junit.Test; | ||
|
@@ -122,4 +144,131 @@ public void testSimpleScan() throws Exception { | |
|
||
testPipeline.run(); | ||
} | ||
|
||
@Test | ||
public void testNameMappingScan() throws Exception { | ||
org.apache.avro.Schema avroSchema = | ||
org.apache.avro.Schema.createRecord( | ||
"test", | ||
null, | ||
null, | ||
false, | ||
ImmutableList.of( | ||
new org.apache.avro.Schema.Field( | ||
"data", org.apache.avro.Schema.create(org.apache.avro.Schema.Type.STRING)), | ||
new org.apache.avro.Schema.Field( | ||
"id", org.apache.avro.Schema.create(org.apache.avro.Schema.Type.LONG)))); | ||
|
||
List<Map<String, Object>> recordData = | ||
ImmutableList.<Map<String, Object>>builder() | ||
.add(ImmutableMap.of("id", 0L, "data", "clarification")) | ||
.add(ImmutableMap.of("id", 1L, "data", "risky")) | ||
.add(ImmutableMap.of("id", 2L, "data", "falafel")) | ||
.build(); | ||
|
||
List<GenericRecord> avroRecords = | ||
recordData.stream() | ||
.map(data -> avroGenericRecord(avroSchema, data)) | ||
.collect(Collectors.toList()); | ||
|
||
Configuration hadoopConf = new Configuration(); | ||
String path = createParquetFile(avroSchema, avroRecords); | ||
HadoopInputFile inputFile = HadoopInputFile.fromLocation(path, hadoopConf); | ||
|
||
NameMapping defaultMapping = NameMapping.of(MappedField.of(1, "id"), MappedField.of(2, "data")); | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. can we include a nested field in this test case? |
||
ImmutableMap<String, String> tableProperties = | ||
ImmutableMap.<String, String>builder() | ||
.put(TableProperties.DEFAULT_NAME_MAPPING, NameMappingParser.toJson(defaultMapping)) | ||
.build(); | ||
|
||
TableIdentifier tableId = | ||
TableIdentifier.of("default", "table" + Long.toString(UUID.randomUUID().hashCode(), 16)); | ||
Table simpleTable = | ||
warehouse | ||
.buildTable(tableId, TestFixtures.SCHEMA) | ||
.withProperties(tableProperties) | ||
.withPartitionSpec(PartitionSpec.unpartitioned()) | ||
.create(); | ||
|
||
MetricsConfig metricsConfig = MetricsConfig.forTable(simpleTable); | ||
Metrics metrics = ParquetUtil.fileMetrics(inputFile, metricsConfig); | ||
DataFile dataFile = | ||
DataFiles.builder(PartitionSpec.unpartitioned()) | ||
.withFormat(FileFormat.PARQUET) | ||
.withInputFile(inputFile) | ||
.withMetrics(metrics) | ||
.build(); | ||
|
||
final Schema beamSchema = IcebergUtils.icebergSchemaToBeamSchema(TestFixtures.SCHEMA); | ||
|
||
simpleTable.newFastAppend().appendFile(dataFile).commit(); | ||
|
||
Map<String, String> catalogProps = | ||
ImmutableMap.<String, String>builder() | ||
.put("type", CatalogUtil.ICEBERG_CATALOG_TYPE_HADOOP) | ||
.put("warehouse", warehouse.location) | ||
.build(); | ||
|
||
IcebergCatalogConfig catalogConfig = | ||
IcebergCatalogConfig.builder() | ||
.setCatalogName("name") | ||
.setCatalogProperties(catalogProps) | ||
.build(); | ||
|
||
PCollection<Row> output = | ||
testPipeline | ||
.apply(IcebergIO.readRows(catalogConfig).from(tableId)) | ||
.apply(ParDo.of(new PrintRow())) | ||
.setCoder(RowCoder.of(beamSchema)); | ||
|
||
final Row[] expectedRows = | ||
recordData.stream() | ||
.map(data -> icebergGenericRecord(TestFixtures.SCHEMA, data)) | ||
.map(record -> IcebergUtils.icebergRecordToBeamRow(beamSchema, record)) | ||
.toArray(Row[]::new); | ||
|
||
PAssert.that(output) | ||
.satisfies( | ||
(Iterable<Row> rows) -> { | ||
assertThat(rows, containsInAnyOrder(expectedRows)); | ||
return null; | ||
}); | ||
|
||
testPipeline.run(); | ||
} | ||
|
||
public static GenericRecord avroGenericRecord( | ||
org.apache.avro.Schema schema, Map<String, Object> values) { | ||
GenericRecord record = new GenericData.Record(schema); | ||
values.forEach(record::put); | ||
return record; | ||
} | ||
|
||
public static Record icebergGenericRecord( | ||
org.apache.iceberg.Schema schema, Map<String, Object> values) { | ||
return org.apache.iceberg.data.GenericRecord.create(schema).copy(values); | ||
} | ||
|
||
public static String createParquetFile(org.apache.avro.Schema schema, List<GenericRecord> records) | ||
throws IOException { | ||
|
||
File tempFile = createTempFile(); | ||
Path file = new Path(tempFile.getPath()); | ||
|
||
AvroParquetWriter.Builder<GenericRecord> builder = AvroParquetWriter.builder(file); | ||
ParquetWriter<GenericRecord> parquetWriter = builder.withSchema(schema).build(); | ||
for (GenericRecord record : records) { | ||
parquetWriter.write(record); | ||
} | ||
parquetWriter.close(); | ||
|
||
return tempFile.getPath(); | ||
} | ||
|
||
private static File createTempFile() throws IOException { | ||
File tempFile = File.createTempFile(ScanSourceTest.class.getSimpleName(), ".tmp"); | ||
tempFile.deleteOnExit(); | ||
boolean unused = tempFile.delete(); | ||
return tempFile; | ||
} | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Oops, something went wrong.
Add this suggestion to a batch that can be applied as a single commit.
This suggestion is invalid because no changes were made to the code.
Suggestions cannot be applied while the pull request is closed.
Suggestions cannot be applied while viewing a subset of changes.
Only one suggestion per line can be applied in a batch.
Add this suggestion to a batch that can be applied as a single commit.
Applying suggestions on deleted lines is not supported.
You must change the existing code in this line in order to create a valid suggestion.
Outdated suggestions cannot be applied.
This suggestion has been applied or marked resolved.
Suggestions cannot be applied from pending reviews.
Suggestions cannot be applied on multi-line comments.
Suggestions cannot be applied while the pull request is queued to merge.
Suggestion cannot be applied right now. Please check back later.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit