|
73 | 73 | import org.apache.arrow.vector.complex.FixedSizeListVector; |
74 | 74 | import org.apache.arrow.vector.complex.ListVector; |
75 | 75 | import org.apache.arrow.vector.complex.MapVector; |
| 76 | +import org.apache.arrow.vector.complex.StructVector; |
76 | 77 | import org.apache.arrow.vector.complex.writer.BaseWriter; |
77 | 78 | import org.apache.arrow.vector.types.DateUnit; |
78 | 79 | import org.apache.arrow.vector.types.FloatingPointPrecision; |
@@ -2517,4 +2518,75 @@ private void compareMaps(Map<String, ?> expected, Map<?, ?> actual) { |
2517 | 2518 | } |
2518 | 2519 | } |
2519 | 2520 | } |
| 2521 | + |
| 2522 | + @Test |
| 2523 | + public void testWriteStruct() throws Exception { |
| 2524 | + |
| 2525 | + // Field definitions |
| 2526 | + FieldType structFieldType = new FieldType(false, new ArrowType.Struct(), null); |
| 2527 | + Field intField = new Field("intField", FieldType.notNullable(new ArrowType.Int(32, true)), null); |
| 2528 | + Field stringField = new Field("stringField", FieldType.notNullable(new ArrowType.Utf8()), null); |
| 2529 | + Field dateField = new Field("dateField", FieldType.notNullable(new ArrowType.Date(DateUnit.DAY)), null); |
| 2530 | + Field structField = new Field("struct", structFieldType, Arrays.asList(intField, stringField, dateField)); |
| 2531 | + |
| 2532 | + // Create empty vector |
| 2533 | + BufferAllocator allocator = new RootAllocator(); |
| 2534 | + StructVector structVector = new StructVector("struct", allocator, structFieldType, null); |
| 2535 | + structVector.initializeChildrenFromFields(Arrays.asList(intField, stringField, dateField)); |
| 2536 | + structVector.allocateNew(); |
| 2537 | + |
| 2538 | + // Set up VSR |
| 2539 | + List<FieldVector> vectors = Arrays.asList(structVector); |
| 2540 | + int rowCount = 3; |
| 2541 | + |
| 2542 | + try (VectorSchemaRoot root = new VectorSchemaRoot(vectors)) { |
| 2543 | + |
| 2544 | + root.setRowCount(rowCount); |
| 2545 | + root.allocateNew(); |
| 2546 | + |
| 2547 | + // Set test data |
| 2548 | + IntVector intVector = (IntVector) structVector.getChild("intField"); |
| 2549 | + VarCharVector stringVector = (VarCharVector) structVector.getChild("stringField"); |
| 2550 | + DateDayVector dateVector = (DateDayVector) structVector.getChild("dateField"); |
| 2551 | + |
| 2552 | + for (int i = 0; i < rowCount; i++) { |
| 2553 | + structVector.setIndexDefined(i); |
| 2554 | + intVector.setSafe(i, i); |
| 2555 | + stringVector.setSafe(i, ("string" + i).getBytes()); |
| 2556 | + dateVector.setSafe(i, (int) LocalDate.now().toEpochDay() + i); |
| 2557 | + } |
| 2558 | + |
| 2559 | + File dataFile = new File(TMP, "testWriteStruct.avro"); |
| 2560 | + |
| 2561 | + // Write an AVRO block using the producer classes |
| 2562 | + try (FileOutputStream fos = new FileOutputStream(dataFile)) { |
| 2563 | + BinaryEncoder encoder = new EncoderFactory().directBinaryEncoder(fos, null); |
| 2564 | + CompositeAvroProducer producer = ArrowToAvroUtils.createCompositeProducer(vectors); |
| 2565 | + for (int row = 0; row < rowCount; row++) { |
| 2566 | + producer.produce(encoder); |
| 2567 | + } |
| 2568 | + encoder.flush(); |
| 2569 | + } |
| 2570 | + |
| 2571 | + // Set up reading the AVRO block as a GenericRecord |
| 2572 | + Schema schema = ArrowToAvroUtils.createAvroSchema(root.getSchema().getFields()); |
| 2573 | + GenericDatumReader<GenericRecord> datumReader = new GenericDatumReader<>(schema); |
| 2574 | + |
| 2575 | + try (InputStream inputStream = new FileInputStream(dataFile)) { |
| 2576 | + |
| 2577 | + BinaryDecoder decoder = DecoderFactory.get().binaryDecoder(inputStream, null); |
| 2578 | + GenericRecord record = null; |
| 2579 | + |
| 2580 | + // Read and check values |
| 2581 | + for (int row = 0; row < rowCount; row++) { |
| 2582 | + record = datumReader.read(record, decoder); |
| 2583 | + assertNotNull(record.get("struct")); |
| 2584 | + GenericRecord structRecord = (GenericRecord) record.get("struct"); |
| 2585 | + assertEquals(row, structRecord.get("intField")); |
| 2586 | + assertEquals("string" + row, structRecord.get("stringField").toString()); |
| 2587 | + assertEquals((int) LocalDate.now().toEpochDay() + row, structRecord.get("dateField")); |
| 2588 | + } |
| 2589 | + } |
| 2590 | + } |
| 2591 | + } |
2520 | 2592 | } |
0 commit comments