Abacn commented on code in PR #36892:
URL: https://github.com/apache/beam/pull/36892#discussion_r2582743500
##########
sdks/java/extensions/avro/src/test/java/org/apache/beam/sdk/extensions/avro/schemas/utils/AvroUtilsTest.java:
##########
@@ -549,6 +550,88 @@ public void testFromBeamSchema() {
assertEquals(getAvroSchema(), avroSchema);
}
+ @Test
+ public void testBeamTimestampNanosLogicalTypeToAvroSchema() {
+ Schema beamSchema =
+ Schema.builder().addLogicalTypeField("timestampNanos",
Timestamp.NANOS).build();
+
+ // Expected Avro schema with timestamp-nanos
+ String expectedJson =
+ "{\"type\": \"record\", \"name\": \"topLevelRecord\", "
+ + "\"fields\": [{\"name\": \"timestampNanos\", "
+ + "\"type\": {\"type\": \"long\", \"logicalType\":
\"timestamp-nanos\"}}]}";
+
+ org.apache.avro.Schema expectedAvroSchema =
+ new org.apache.avro.Schema.Parser().parse(expectedJson);
+
+ assertEquals(expectedAvroSchema, AvroUtils.toAvroSchema(beamSchema));
+ }
+
+ @Test
+ public void testBeamTimestampNanosToGenericRecord() {
+ Schema beamSchema =
+ Schema.builder().addLogicalTypeField("timestampNanos",
Timestamp.NANOS).build();
+
+ java.time.Instant instant =
java.time.Instant.parse("2000-01-01T01:02:03.123456789Z");
+ Row beamRow = Row.withSchema(beamSchema).addValue(instant).build();
+
+ // Expected nanos since epoch
+ long expectedNanos = TimeUnit.SECONDS.toNanos(instant.getEpochSecond()) +
instant.getNano();
+
+ org.apache.avro.Schema avroSchema = AvroUtils.toAvroSchema(beamSchema);
+ GenericRecord avroRecord = AvroUtils.toGenericRecord(beamRow, avroSchema);
+
+ assertEquals(expectedNanos, avroRecord.get("timestampNanos"));
+ }
+
+ @Test
+ public void testTimestampNanosRoundTrip() {
+ Schema beamSchema =
+ Schema.builder().addLogicalTypeField("timestampNanos",
Timestamp.NANOS).build();
+
+ // Test various nanosecond precisions
+ java.time.Instant[] testInstants = {
+ java.time.Instant.parse("2000-01-01T00:00:00.000000001Z"), // 1 nano
+ java.time.Instant.parse("2000-01-01T00:00:00.123456789Z"), // full nanos
+ java.time.Instant.parse("2000-01-01T00:00:00.999999999Z"), // max nanos
+ java.time.Instant.ofEpochSecond(0L, Long.MAX_VALUE), // max supported
+ java.time.Instant.ofEpochSecond(0L, Long.MIN_VALUE), // min supported
Review Comment:
consider a test case of negative long value compared to epoch.
##########
sdks/java/extensions/avro/src/main/java/org/apache/beam/sdk/extensions/avro/schemas/utils/AvroUtils.java:
##########
@@ -1340,6 +1353,15 @@ private static org.apache.avro.Schema getFieldSchema(
java.time.Instant instant = (java.time.Instant) value;
return TimeUnit.SECONDS.toMicros(instant.getEpochSecond())
+ TimeUnit.NANOSECONDS.toMicros(instant.getNano());
+ } else if (Timestamp.IDENTIFIER.equals(identifier)) {
+ java.time.Instant instant = (java.time.Instant) value;
+ // Use BigInteger to work around long overflows so that minimum
timestamp can be
Review Comment:
How could an overflow be possible? `instant.getEpochSecond()` returns a
long. And in the below it converted back to long via `longValueExact`.
It seems if there were an overflow it would still not work under current
code.
--
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]