[ https://issues.apache.org/jira/browse/BEAM-4076?focusedWorklogId=123361&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-123361 ]
ASF GitHub Bot logged work on BEAM-4076: ---------------------------------------- Author: ASF GitHub Bot Created on: 14/Jul/18 22:53 Start Date: 14/Jul/18 22:53 Worklog Time Spent: 10m Work Description: reuvenlax closed pull request #5953: [BEAM-4076] Fix bugs in generated schema code URL: https://github.com/apache/beam/pull/5953 This is a PR merged from a forked repository. As GitHub hides the original diff on merge, it is displayed below for the sake of provenance: As this is a foreign pull request (from a fork), the diff is supplied below (as it won't show otherwise due to GitHub magic): diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/GetterBasedSchemaProvider.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/GetterBasedSchemaProvider.java index 108c74601ce..7d0856bab17 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/GetterBasedSchemaProvider.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/GetterBasedSchemaProvider.java @@ -26,6 +26,7 @@ import java.lang.reflect.Type; import java.util.List; import java.util.Map; +import javax.annotation.Nullable; import org.apache.beam.sdk.annotations.Experimental; import org.apache.beam.sdk.annotations.Experimental.Kind; import org.apache.beam.sdk.schemas.Schema.FieldType; @@ -108,8 +109,13 @@ } @SuppressWarnings("unchecked") + @Nullable private <T> T fromValue( FieldType type, T value, Type fieldType, Type elemenentType, Type keyType, Type valueType) { + if (value == null) { + return null; + } + if (TypeName.ROW.equals(type.getTypeName())) { return (T) fromRow((Row) value, (Class) fieldType); } else if (TypeName.ARRAY.equals(type.getTypeName())) { diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/ByteBuddyUtils.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/ByteBuddyUtils.java index 1593fbb5fab..cdbb7c71af7 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/ByteBuddyUtils.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/ByteBuddyUtils.java @@ -51,8 +51,7 @@ import org.apache.beam.sdk.values.reflect.FieldValueSetter; import org.apache.commons.lang3.ArrayUtils; import org.apache.commons.lang3.ClassUtils; -import org.joda.time.DateTime; -import org.joda.time.ReadableDateTime; +import org.joda.time.Instant; import org.joda.time.ReadableInstant; class ByteBuddyUtils { @@ -61,7 +60,7 @@ private static final ForLoadedType BYTE_ARRAY_TYPE = new ForLoadedType(byte[].class); private static final ForLoadedType BYTE_BUFFER_TYPE = new ForLoadedType(ByteBuffer.class); private static final ForLoadedType CHAR_SEQUENCE_TYPE = new ForLoadedType(CharSequence.class); - private static final ForLoadedType DATE_TIME_TYPE = new ForLoadedType(DateTime.class); + private static final ForLoadedType INSTANT_TYPE = new ForLoadedType(Instant.class); private static final ForLoadedType LIST_TYPE = new ForLoadedType(List.class); private static final ForLoadedType READABLE_INSTANT_TYPE = new ForLoadedType(ReadableInstant.class); @@ -166,7 +165,7 @@ protected Type convertMap(TypeDescriptor<?> type) { @Override protected Type convertDateTime(TypeDescriptor<?> type) { - return ReadableInstant.class; + return Instant.class; } @Override @@ -258,16 +257,16 @@ protected StackManipulation convertMap(TypeDescriptor<?> type) { @Override protected StackManipulation convertDateTime(TypeDescriptor<?> type) { - // If the class type is a ReadableDateTime, then return it. - if (ReadableDateTime.class.isAssignableFrom(type.getRawType())) { + // If the class type is an Instant, then return it. + if (Instant.class.isAssignableFrom(type.getRawType())) { return readValue; } // Otherwise, generate the following code: - // return new DateTime(value.getMillis()); + // return new Instant(value.getMillis()); return new StackManipulation.Compound( // Create a new instance of the target type. - TypeCreation.of(DATE_TIME_TYPE), + TypeCreation.of(INSTANT_TYPE), Duplication.SINGLE, readValue, TypeCasting.to(READABLE_INSTANT_TYPE), @@ -279,7 +278,7 @@ protected StackManipulation convertDateTime(TypeDescriptor<?> type) { .getOnly()), // Construct a DateTime object containing the millis. MethodInvocation.invoke( - DATE_TIME_TYPE + INSTANT_TYPE .getDeclaredMethods() .filter( ElementMatchers.isConstructor() diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtils.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtils.java index f3dc5706f0a..e819bdc8520 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtils.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtils.java @@ -94,7 +94,15 @@ private static void validateJavaBean( } if (!type.equals(setterType)) { throw new RuntimeException( - "JavaBean contained mismatching setter for field" + type.getName()); + "JavaBean contained setter for field " + + type.getName() + + " that had a mismatching type."); + } + if (!type.isNullable() == setterType.isNullable()) { + throw new RuntimeException( + "JavaBean contained setter for field " + + type.getName() + + " that had a mismatching nullable attribute."); } } } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/StaticSchemaInference.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/StaticSchemaInference.java index d0b315f0706..38fa42aa96d 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/StaticSchemaInference.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/schemas/utils/StaticSchemaInference.java @@ -81,7 +81,7 @@ public static TypeInformation forField(Field field) { return new TypeInformation( field.getName(), TypeDescriptor.of(field.getGenericType()), - field.getAnnotation(Nullable.class) != null); + field.isAnnotationPresent(Nullable.class)); } /** Construct a {@link TypeInformation} from a class getter. */ @@ -95,7 +95,7 @@ public static TypeInformation forGetter(Method method) { throw new RuntimeException("Getter has wrong prefix " + method.getName()); } TypeDescriptor type = TypeDescriptor.of(method.getGenericReturnType()); - boolean nullable = method.getAnnotation(Nullable.class) != null; + boolean nullable = method.isAnnotationPresent(Nullable.class); return new TypeInformation(name, type, nullable); } @@ -159,8 +159,12 @@ public static Schema schemaFromClass( Class<?> clazz, Function<Class, List<TypeInformation>> getTypesForClass) { Schema.Builder builder = Schema.builder(); for (TypeInformation type : getTypesForClass.apply(clazz)) { - // TODO: look for nullable annotation. - builder.addField(type.getName(), fieldFromType(type.getType(), getTypesForClass)); + Schema.FieldType fieldType = fieldFromType(type.getType(), getTypesForClass); + if (type.isNullable()) { + builder.addNullableField(type.getName(), fieldType); + } else { + builder.addField(type.getName(), fieldType); + } } return builder.build(); } diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/values/RowWithGetters.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/values/RowWithGetters.java index f1d737037ac..ebe1c31cedc 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/values/RowWithGetters.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/values/RowWithGetters.java @@ -25,6 +25,7 @@ import java.util.stream.Collectors; import javax.annotation.Nullable; import org.apache.beam.sdk.schemas.Schema; +import org.apache.beam.sdk.schemas.Schema.Field; import org.apache.beam.sdk.schemas.Schema.FieldType; import org.apache.beam.sdk.schemas.Schema.TypeName; import org.apache.beam.sdk.values.reflect.FieldValueGetter; @@ -56,10 +57,13 @@ @Override @SuppressWarnings({"TypeParameterUnusedInFormals", "unchecked"}) public <T> T getValue(int fieldIdx) { - FieldType type = getSchema().getField(fieldIdx).getType(); + Field field = getSchema().getField(fieldIdx); + FieldType type = field.getType(); Object fieldValue = getters.get(fieldIdx).get(getterTarget); - - return getValue(type, fieldValue, fieldIdx); + if (fieldValue == null && !field.getNullable()) { + throw new RuntimeException("Null value set on non-nullable field" + field); + } + return fieldValue != null ? getValue(type, fieldValue, fieldIdx) : null; } private List getListValue(FieldType elementType, Object fieldValue) { diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaBeanSchemaTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaBeanSchemaTest.java index 7ed115a3c57..9ef7f82e2ba 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaBeanSchemaTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaBeanSchemaTest.java @@ -106,8 +106,8 @@ public void testToRow() throws NoSuchSchemaException { assertEquals((int) 3, row.getInt32("anInt")); assertEquals((long) 4, row.getInt64("aLong")); assertEquals(true, row.getBoolean("aBoolean")); - assertEquals(DATE, row.getDateTime("dateTime")); - assertEquals(DATE, row.getDateTime("instant")); + assertEquals(DATE.toInstant(), row.getDateTime("dateTime")); + assertEquals(DATE.toInstant(), row.getDateTime("instant")); assertArrayEquals(BYTE_ARRAY, row.getBytes("bytes")); assertArrayEquals(BYTE_ARRAY, row.getBytes("byteBuffer")); assertEquals(BigDecimal.ONE, row.getDecimal("bigDecimal")); @@ -160,8 +160,8 @@ public void testRecursiveGetters() throws NoSuchSchemaException { assertEquals((int) 3, nestedRow.getInt32("anInt")); assertEquals((long) 4, nestedRow.getInt64("aLong")); assertEquals(true, nestedRow.getBoolean("aBoolean")); - assertEquals(DATE, nestedRow.getDateTime("dateTime")); - assertEquals(DATE, nestedRow.getDateTime("instant")); + assertEquals(DATE.toInstant(), nestedRow.getDateTime("dateTime")); + assertEquals(DATE.toInstant(), nestedRow.getDateTime("instant")); assertArrayEquals("not equal", BYTE_ARRAY, nestedRow.getBytes("bytes")); assertArrayEquals("not equal", BYTE_ARRAY, nestedRow.getBytes("byteBuffer")); assertEquals(BigDecimal.ONE, nestedRow.getDecimal("bigDecimal")); diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaFieldSchemaTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaFieldSchemaTest.java index 314914ac088..c344c4aa0c5 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaFieldSchemaTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/JavaFieldSchemaTest.java @@ -21,11 +21,14 @@ import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NESTED_ARRAYS_POJO_SCHEMA; import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NESTED_ARRAY_POJO_SCHEMA; import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NESTED_MAP_POJO_SCHEMA; +import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NESTED_NULLABLE_SCHEMA; import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NESTED_POJO_SCHEMA; +import static org.apache.beam.sdk.schemas.utils.TestPOJOs.NULLABLES_SCHEMA; import static org.apache.beam.sdk.schemas.utils.TestPOJOs.PRIMITIVE_ARRAY_POJO_SCHEMA; import static org.apache.beam.sdk.schemas.utils.TestPOJOs.SIMPLE_POJO_SCHEMA; import static org.junit.Assert.assertArrayEquals; import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertNull; import static org.junit.Assert.assertSame; import com.google.common.collect.ImmutableList; @@ -43,6 +46,8 @@ import org.apache.beam.sdk.schemas.utils.TestPOJOs.NestedArraysPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.NestedMapPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.NestedPOJO; +import org.apache.beam.sdk.schemas.utils.TestPOJOs.POJOWithNestedNullable; +import org.apache.beam.sdk.schemas.utils.TestPOJOs.POJOWithNullables; import org.apache.beam.sdk.schemas.utils.TestPOJOs.PrimitiveArrayPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.SimplePOJO; import org.apache.beam.sdk.values.Row; @@ -112,7 +117,7 @@ public void testToRow() throws NoSuchSchemaException { assertEquals((int) 3, row.getInt32("anInt")); assertEquals((long) 4, row.getInt64("aLong")); assertEquals(true, row.getBoolean("aBoolean")); - assertEquals(DATE, row.getDateTime("dateTime")); + assertEquals(DATE.toInstant(), row.getDateTime("dateTime")); assertEquals(INSTANT, row.getDateTime("instant").toInstant()); assertArrayEquals(BYTE_ARRAY, row.getBytes("bytes")); assertArrayEquals(BYTE_BUFFER.array(), row.getBytes("byteBuffer")); @@ -166,7 +171,7 @@ public void testRecursiveGetters() throws NoSuchSchemaException { assertEquals((int) 3, nestedRow.getInt32("anInt")); assertEquals((long) 4, nestedRow.getInt64("aLong")); assertEquals(true, nestedRow.getBoolean("aBoolean")); - assertEquals(DATE, nestedRow.getDateTime("dateTime")); + assertEquals(DATE.toInstant(), nestedRow.getDateTime("dateTime")); assertEquals(INSTANT, nestedRow.getDateTime("instant").toInstant()); assertArrayEquals("not equal", BYTE_ARRAY, nestedRow.getBytes("bytes")); assertArrayEquals("not equal", BYTE_BUFFER.array(), nestedRow.getBytes("byteBuffer")); @@ -341,4 +346,45 @@ public void testMapFieldSetters() throws NoSuchSchemaException { assertEquals("string2", pojo.map.get("simple2").str); assertEquals("string3", pojo.map.get("simple3").str); } + + @Test + public void testNullValuesGetters() throws NoSuchSchemaException { + SchemaRegistry registry = SchemaRegistry.createDefault(); + + Row row = + registry.getToRowFunction(POJOWithNullables.class).apply(new POJOWithNullables(null, 42)); + assertNull(row.getString("str")); + assertEquals(42, row.getInt32("anInt")); + } + + @Test + public void testNullValuesSetters() throws NoSuchSchemaException { + SchemaRegistry registry = SchemaRegistry.createDefault(); + + Row row = Row.withSchema(NULLABLES_SCHEMA).addValues(null, 42).build(); + POJOWithNullables pojo = registry.getFromRowFunction(POJOWithNullables.class).apply(row); + assertNull(pojo.str); + assertEquals(42, pojo.anInt); + } + + @Test + public void testNestedNullValuesGetters() throws NoSuchSchemaException { + SchemaRegistry registry = SchemaRegistry.createDefault(); + + Row row = + registry + .getToRowFunction(POJOWithNestedNullable.class) + .apply(new POJOWithNestedNullable(null)); + assertNull(row.getValue("nested")); + } + + @Test + public void testNNestedullValuesSetters() throws NoSuchSchemaException { + SchemaRegistry registry = SchemaRegistry.createDefault(); + + Row row = Row.withSchema(NESTED_NULLABLE_SCHEMA).addValue(null).build(); + POJOWithNestedNullable pojo = + registry.getFromRowFunction(POJOWithNestedNullable.class).apply(row); + assertNull(pojo.nested); + } } diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtilsTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtilsTest.java index 72c423822f1..c18d3f6adb8 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtilsTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/JavaBeanUtilsTest.java @@ -29,6 +29,8 @@ import static org.apache.beam.sdk.schemas.utils.TestJavaBeans.SIMPLE_BEAN_SCHEMA; import static org.junit.Assert.assertArrayEquals; import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertFalse; +import static org.junit.Assert.assertTrue; import java.math.BigDecimal; import java.nio.ByteBuffer; @@ -37,20 +39,38 @@ import org.apache.beam.sdk.schemas.Schema; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.BeanWithBoxedFields; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.BeanWithByteArray; +import org.apache.beam.sdk.schemas.utils.TestJavaBeans.MismatchingNullableBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.NestedArrayBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.NestedBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.NestedCollectionBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.NestedMapBean; +import org.apache.beam.sdk.schemas.utils.TestJavaBeans.NullableBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.PrimitiveArrayBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.PrimitiveMapBean; import org.apache.beam.sdk.schemas.utils.TestJavaBeans.SimpleBean; import org.apache.beam.sdk.values.reflect.FieldValueGetter; import org.apache.beam.sdk.values.reflect.FieldValueSetter; import org.joda.time.DateTime; +import org.junit.Rule; import org.junit.Test; +import org.junit.rules.ExpectedException; /** Tests for the {@link JavaBeanUtils} class. */ public class JavaBeanUtilsTest { + @Rule public ExpectedException thrown = ExpectedException.none(); + + @Test + public void testNullable() { + Schema schema = JavaBeanUtils.schemaFromJavaBeanClass(NullableBean.class); + assertTrue(schema.getField("str").getNullable()); + assertFalse(schema.getField("anInt").getNullable()); + } + + @Test + public void testMismatchingNullable() { + thrown.expect(RuntimeException.class); + Schema schema = JavaBeanUtils.schemaFromJavaBeanClass(MismatchingNullableBean.class); + } @Test public void testSimpleBean() { @@ -120,8 +140,8 @@ public void testGeneratedSimpleGetters() { assertEquals((int) 43, getters.get(3).get(simpleBean)); assertEquals((long) 44, getters.get(4).get(simpleBean)); assertEquals(true, getters.get(5).get(simpleBean)); - assertEquals(DateTime.parse("1979-03-14"), getters.get(6).get(simpleBean)); - assertEquals(DateTime.parse("1979-03-15"), getters.get(7).get(simpleBean)); + assertEquals(DateTime.parse("1979-03-14").toInstant(), getters.get(6).get(simpleBean)); + assertEquals(DateTime.parse("1979-03-15").toInstant(), getters.get(7).get(simpleBean)); assertArrayEquals( "Unexpected bytes", "bytes1".getBytes(Charset.defaultCharset()), @@ -146,8 +166,8 @@ public void testGeneratedSimpleSetters() { setters.get(3).set(simpleBean, (int) 43); setters.get(4).set(simpleBean, (long) 44); setters.get(5).set(simpleBean, true); - setters.get(6).set(simpleBean, DateTime.parse("1979-03-14")); - setters.get(7).set(simpleBean, DateTime.parse("1979-03-15")); + setters.get(6).set(simpleBean, DateTime.parse("1979-03-14").toInstant()); + setters.get(7).set(simpleBean, DateTime.parse("1979-03-15").toInstant()); setters.get(8).set(simpleBean, "bytes1".getBytes(Charset.defaultCharset())); setters.get(9).set(simpleBean, "bytes2".getBytes(Charset.defaultCharset())); setters.get(10).set(simpleBean, new BigDecimal(42)); diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/POJOUtilsTest.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/POJOUtilsTest.java index fb80d9c61e9..99fd2454275 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/POJOUtilsTest.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/POJOUtilsTest.java @@ -29,6 +29,8 @@ import static org.apache.beam.sdk.schemas.utils.TestPOJOs.SIMPLE_POJO_SCHEMA; import static org.junit.Assert.assertArrayEquals; import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertFalse; +import static org.junit.Assert.assertTrue; import java.math.BigDecimal; import java.nio.ByteBuffer; @@ -41,6 +43,7 @@ import org.apache.beam.sdk.schemas.utils.TestPOJOs.NestedPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.POJOWithBoxedFields; import org.apache.beam.sdk.schemas.utils.TestPOJOs.POJOWithByteArray; +import org.apache.beam.sdk.schemas.utils.TestPOJOs.POJOWithNullables; import org.apache.beam.sdk.schemas.utils.TestPOJOs.PrimitiveArrayPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.PrimitiveMapPOJO; import org.apache.beam.sdk.schemas.utils.TestPOJOs.SimplePOJO; @@ -58,6 +61,13 @@ static final ByteBuffer BYTE_BUFFER = ByteBuffer.wrap("byteBuffer".getBytes(Charset.defaultCharset())); + @Test + public void testNullables() { + Schema schema = POJOUtils.schemaFromPojoClass(POJOWithNullables.class); + assertTrue(schema.getField("str").getNullable()); + assertFalse(schema.getField("anInt").getNullable()); + } + @Test public void testSimplePOJO() { Schema schema = POJOUtils.schemaFromPojoClass(SimplePOJO.class); @@ -126,8 +136,8 @@ public void testGeneratedSimpleGetters() { assertEquals((int) 43, getters.get(3).get(simplePojo)); assertEquals((long) 44, getters.get(4).get(simplePojo)); assertEquals(true, getters.get(5).get(simplePojo)); - assertEquals(DATE, getters.get(6).get(simplePojo)); - assertEquals(INSTANT, ((DateTime) getters.get(7).get(simplePojo)).toInstant()); + assertEquals(DATE.toInstant(), getters.get(6).get(simplePojo)); + assertEquals(INSTANT, getters.get(7).get(simplePojo)); assertArrayEquals("Unexpected bytes", BYTE_ARRAY, (byte[]) getters.get(8).get(simplePojo)); assertArrayEquals( "Unexpected bytes", BYTE_BUFFER.array(), (byte[]) getters.get(9).get(simplePojo)); @@ -147,7 +157,7 @@ public void testGeneratedSimpleSetters() { setters.get(3).set(simplePojo, (int) 43); setters.get(4).set(simplePojo, (long) 44); setters.get(5).set(simplePojo, true); - setters.get(6).set(simplePojo, DATE); + setters.get(6).set(simplePojo, DATE.toInstant()); setters.get(7).set(simplePojo, INSTANT); setters.get(8).set(simplePojo, BYTE_ARRAY); setters.get(9).set(simplePojo, BYTE_BUFFER.array()); diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestJavaBeans.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestJavaBeans.java index 92c1cb3d1fc..febe67496d4 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestJavaBeans.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestJavaBeans.java @@ -22,6 +22,7 @@ import java.nio.ByteBuffer; import java.util.List; import java.util.Map; +import javax.annotation.Nullable; import org.apache.beam.sdk.schemas.DefaultSchema; import org.apache.beam.sdk.schemas.JavaBeanSchema; import org.apache.beam.sdk.schemas.Schema; @@ -31,6 +32,49 @@ /** Various Java Beans and associated schemas used in tests. */ public class TestJavaBeans { + /** A Bean containing one nullable and one non-nullable type. */ + @DefaultSchema(JavaBeanSchema.class) + public static class NullableBean { + @Nullable private String str; + private int anInt; + + public NullableBean() {} + + @Nullable + public String getStr() { + return str; + } + + public void setStr(@Nullable String str) { + this.str = str; + } + + public int getAnInt() { + return anInt; + } + + public void setAnInt(int anInt) { + this.anInt = anInt; + } + } + + /** A Bean containing nullable getter but a non-nullable setter. */ + @DefaultSchema(JavaBeanSchema.class) + public static class MismatchingNullableBean { + @Nullable private String str; + + public MismatchingNullableBean() {} + + @Nullable + public String getStr() { + return str; + } + + public void setStr(String str) { + this.str = str; + } + } + /** A simple Bean containing basic types. * */ @DefaultSchema(JavaBeanSchema.class) public static class SimpleBean { diff --git a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestPOJOs.java b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestPOJOs.java index 5d60884a2ab..9d03f3fe613 100644 --- a/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestPOJOs.java +++ b/sdks/java/core/src/test/java/org/apache/beam/sdk/schemas/utils/TestPOJOs.java @@ -22,6 +22,7 @@ import java.nio.ByteBuffer; import java.util.List; import java.util.Map; +import javax.annotation.Nullable; import org.apache.beam.sdk.schemas.DefaultSchema; import org.apache.beam.sdk.schemas.JavaFieldSchema; import org.apache.beam.sdk.schemas.Schema; @@ -31,6 +32,40 @@ /** Various Java POJOs and associated schemas used in tests. */ public class TestPOJOs { + /** A POJO containing one nullable and one non-nullable type. */ + @DefaultSchema(JavaFieldSchema.class) + public static class POJOWithNullables { + @Nullable public String str; + public int anInt; + + public POJOWithNullables(@Nullable String str, int anInt) { + this.str = str; + this.anInt = anInt; + } + + public POJOWithNullables() {} + } + + /** The schema for {@link POJOWithNullables}. * */ + public static final Schema NULLABLES_SCHEMA = + Schema.builder().addNullableField("str", FieldType.STRING).addInt32Field("anInt").build(); + + /** a POJO containing a nested nullable field. * */ + @DefaultSchema(JavaFieldSchema.class) + public static class POJOWithNestedNullable { + @Nullable public POJOWithNullables nested; + + public POJOWithNestedNullable(@Nullable POJOWithNullables nested) { + this.nested = nested; + } + + public POJOWithNestedNullable() {} + } + + /** The schema for {@link POJOWithNestedNullable}. * */ + public static final Schema NESTED_NULLABLE_SCHEMA = + Schema.builder().addNullableField("nested", FieldType.row(NULLABLES_SCHEMA)).build(); + /** A simple POJO containing basic types. * */ @DefaultSchema(JavaFieldSchema.class) public static class SimplePOJO { ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org Issue Time Tracking ------------------- Worklog Id: (was: 123361) Time Spent: 11h 50m (was: 11h 40m) > Schema followups > ---------------- > > Key: BEAM-4076 > URL: https://issues.apache.org/jira/browse/BEAM-4076 > Project: Beam > Issue Type: Improvement > Components: beam-model, dsl-sql, sdk-java-core > Reporter: Kenneth Knowles > Priority: Major > Time Spent: 11h 50m > Remaining Estimate: 0h > > This umbrella bug contains subtasks with followups for Beam schemas, which > were moved from SQL to the core Java SDK and made to be type-name-based > rather than coder based. -- This message was sent by Atlassian JIRA (v7.6.3#76005)