aihuaxu commented on code in PR #11415: URL: https://github.com/apache/iceberg/pull/11415#discussion_r1885104029
########## core/src/main/java/org/apache/iceberg/variants/Variants.java: ########## @@ -0,0 +1,276 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.iceberg.variants; + +import java.math.BigDecimal; +import java.nio.ByteBuffer; +import java.util.List; +import java.util.Map; +import org.apache.iceberg.util.DateTimeUtil; + +public class Variants { + private Variants() {} + + enum LogicalType { Review Comment: I was thinking of simplifying the types to BasicType and PrimitiveType only here. LogicalType and PhysicalType may not needed? But we can revisit later if needed. > enum BasicType { > PRIMITIVE(0), > SHORT_STRING(1), > OBJECT(2), > ARRAY(3); > } and public enum PrimitiveType { NULL(0), TRUE(1), FALSE(2), ... } ########## core/src/main/java/org/apache/iceberg/variants/ShreddedObject.java: ########## @@ -0,0 +1,211 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.iceberg.variants; + +import java.nio.ByteBuffer; +import java.nio.ByteOrder; +import java.util.Map; +import org.apache.iceberg.relocated.com.google.common.base.Preconditions; +import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap; +import org.apache.iceberg.relocated.com.google.common.collect.Maps; +import org.apache.iceberg.util.Pair; +import org.apache.iceberg.util.SortedMerge; + +/** + * A variant Object that handles full or partial shredding. + * + * <p>Metadata stored for an object must be the same regardless of whether the object is shredded. + * This class assumes that the metadata from the unshredded object can be used for the shredded + * fields. This also does not allow updating or replacing the metadata for the unshredded object, + * which could require recursively rewriting field IDs. + */ +class ShreddedObject implements VariantObject { + private final SerializedMetadata metadata; + private final SerializedObject unshredded; + private final Map<String, VariantValue> shreddedFields = Maps.newHashMap(); + private SerializationState serializationState = null; + + ShreddedObject(SerializedMetadata metadata) { + this.metadata = metadata; + this.unshredded = null; + } + + ShreddedObject(SerializedObject unshredded) { + this.metadata = unshredded.metadata(); + this.unshredded = unshredded; + } + + public void put(String field, VariantValue value) { + Preconditions.checkArgument( + metadata.id(field) >= 0, "Cannot find field name in metadata: %s", field); + + // allow setting fields that are contained in unshredded. this avoids read-time failures and + // simplifies replacing field values. + shreddedFields.put(field, value); + this.serializationState = null; + } + + @Override + public VariantValue get(String field) { Review Comment: I see what we are implementing in ShreddedObject: basically we are providing the same interface get(String field) as regular VariantObject. Given the following example, assume `event.location.latitude` is shredded while `event.location.longitude` is not. How do we model shredded `event` object - where to place the field `location` ? > event { > event_id; > location { > latitude; > longitude; > } > } ########## core/src/test/java/org/apache/iceberg/variants/TestShreddedObject.java: ########## @@ -0,0 +1,446 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.iceberg.variants; + +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.math.BigDecimal; +import java.nio.ByteBuffer; +import java.nio.ByteOrder; +import java.util.Map; +import java.util.Random; +import java.util.Set; +import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap; +import org.apache.iceberg.relocated.com.google.common.collect.Maps; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.apache.iceberg.util.DateTimeUtil; +import org.apache.iceberg.util.Pair; +import org.apache.iceberg.util.RandomUtil; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; + +public class TestShreddedObject { + private static final Map<String, VariantValue> FIELDS = + ImmutableMap.of( + "a", + Variants.of(34), + "b", + Variants.of("iceberg"), + "c", + Variants.of(new BigDecimal("12.21"))); Review Comment: In our ShreddedObject model, the field values can be a VariantObject as well which could be a ShreddedObject, right? Can we add such coverage? ########## core/src/test/java/org/apache/iceberg/variants/TestShreddedObject.java: ########## @@ -0,0 +1,446 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.iceberg.variants; + +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.math.BigDecimal; +import java.nio.ByteBuffer; +import java.nio.ByteOrder; +import java.util.Map; +import java.util.Random; +import java.util.Set; +import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap; +import org.apache.iceberg.relocated.com.google.common.collect.Maps; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.apache.iceberg.util.DateTimeUtil; +import org.apache.iceberg.util.Pair; +import org.apache.iceberg.util.RandomUtil; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; + +public class TestShreddedObject { + private static final Map<String, VariantValue> FIELDS = + ImmutableMap.of( + "a", + Variants.of(34), + "b", + Variants.of("iceberg"), + "c", + Variants.of(new BigDecimal("12.21"))); + + private final Random random = new Random(871925); + + @Test + public void testShreddedFields() { + ShreddedObject object = createShreddedObject(FIELDS).second(); + + assertThat(object.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testShreddedSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripMinimalBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testShreddedSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripLargeBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testUnshreddedObjectSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripMinimalBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testUnshreddedObjectSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripLargeBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testPartiallyShreddedObjectReplacement() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + assertThat(partial.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(partial.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(partial.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(partial.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testPartiallyShreddedObjectGetMissingField() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // missing fields are returned as null + assertThat(partial.get("d")).isNull(); + } + + @Test + public void testPartiallyShreddedObjectPutMissingFieldFailure() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // d is not defined in the variant metadata and will fail + assertThatThrownBy(() -> partial.put("d", Variants.ofIsoDate("2024-10-12"))) + .isInstanceOf(IllegalArgumentException.class) + .hasMessage("Cannot find field name in metadata: d"); + } + + @Test + public void testPartiallyShreddedObjectSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject partial = pair.second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + VariantValue value = roundTripMinimalBuffer(partial, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(actual.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testPartiallyShreddedObjectSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject partial = pair.second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + VariantValue value = roundTripLargeBuffer(partial, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(actual.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testTwoByteOffsets() { + // a string larger than 255 bytes to push the value offset size above 1 byte + String randomString = RandomUtil.generateString(300, random); + SerializedPrimitive bigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("big", bigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("big").asPrimitive().get()).isEqualTo(randomString); + } + + @Test + public void testThreeByteOffsets() { + // a string larger than 65535 bytes to push the value offset size above 2 bytes + String randomString = RandomUtil.generateString(70_000, random); + SerializedPrimitive reallyBigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("really-big", reallyBigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("really-big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("really-big").asPrimitive().get()).isEqualTo(randomString); + } + + @Test + public void testFourByteOffsets() { + // a string larger than 16777215 bytes to push the value offset size above 3 bytes + String randomString = RandomUtil.generateString(16_777_300, random); + SerializedPrimitive reallyBigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("really-big", reallyBigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("really-big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("really-big").asPrimitive().get()).isEqualTo(randomString); + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + @SuppressWarnings({"unchecked", "rawtypes"}) + public void testLargeObject(boolean sortFieldNames) { + Map<String, VariantPrimitive<String>> fields = Maps.newHashMap(); + for (int i = 0; i < 10_000; i += 1) { + fields.put( + RandomUtil.generateString(10, random), + Variants.of(RandomUtil.generateString(10, random))); + } + + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(fields.keySet(), sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, (Map) fields); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(10_000); + + for (Map.Entry<String, VariantPrimitive<String>> entry : fields.entrySet()) { + VariantValue fieldValue = object.get(entry.getKey()); + assertThat(fieldValue.type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(fieldValue.asPrimitive().get()).isEqualTo(entry.getValue().get()); + } + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testTwoByteFieldIds(boolean sortFieldNames) { + Set<String> keySet = Sets.newHashSet(); + for (int i = 0; i < 10_000; i += 1) { + keySet.add(RandomUtil.generateString(10, random)); + } + + Map<String, VariantValue> data = + ImmutableMap.of("aa", FIELDS.get("a"), "AA", FIELDS.get("b"), "ZZ", FIELDS.get("c")); + + // create metadata from the large key set and the actual keys + keySet.addAll(data.keySet()); + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(keySet, sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, data); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(3); + + assertThat(object.get("aa").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("aa").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("AA").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("AA").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("ZZ").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("ZZ").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testThreeByteFieldIds(boolean sortFieldNames) { + Set<String> keySet = Sets.newHashSet(); + for (int i = 0; i < 100_000; i += 1) { + keySet.add(RandomUtil.generateString(10, random)); + } + + Map<String, VariantValue> data = + ImmutableMap.of("aa", FIELDS.get("a"), "AA", FIELDS.get("b"), "ZZ", FIELDS.get("c")); + + // create metadata from the large key set and the actual keys + keySet.addAll(data.keySet()); + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(keySet, sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, data); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(3); + + assertThat(object.get("aa").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("aa").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("AA").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("AA").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("ZZ").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("ZZ").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + static VariantValue roundTripMinimalBuffer(ShreddedObject object, SerializedMetadata metadata) { + ByteBuffer serialized = + ByteBuffer.allocate(object.sizeInBytes()).order(ByteOrder.LITTLE_ENDIAN); + object.writeTo(serialized, 0); + + return Variants.from(metadata, serialized); + } + + static VariantValue roundTripLargeBuffer(ShreddedObject object, SerializedMetadata metadata) { Review Comment: Same here. nit: private scope? ########## core/src/test/java/org/apache/iceberg/variants/TestShreddedObject.java: ########## @@ -0,0 +1,446 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.iceberg.variants; + +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.math.BigDecimal; +import java.nio.ByteBuffer; +import java.nio.ByteOrder; +import java.util.Map; +import java.util.Random; +import java.util.Set; +import org.apache.iceberg.relocated.com.google.common.collect.ImmutableMap; +import org.apache.iceberg.relocated.com.google.common.collect.Maps; +import org.apache.iceberg.relocated.com.google.common.collect.Sets; +import org.apache.iceberg.util.DateTimeUtil; +import org.apache.iceberg.util.Pair; +import org.apache.iceberg.util.RandomUtil; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.params.ParameterizedTest; +import org.junit.jupiter.params.provider.ValueSource; + +public class TestShreddedObject { + private static final Map<String, VariantValue> FIELDS = + ImmutableMap.of( + "a", + Variants.of(34), + "b", + Variants.of("iceberg"), + "c", + Variants.of(new BigDecimal("12.21"))); + + private final Random random = new Random(871925); + + @Test + public void testShreddedFields() { + ShreddedObject object = createShreddedObject(FIELDS).second(); + + assertThat(object.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testShreddedSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripMinimalBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testShreddedSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripLargeBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testUnshreddedObjectSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripMinimalBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testUnshreddedObjectSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject object = pair.second(); + + VariantValue value = roundTripLargeBuffer(object, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.numElements()).isEqualTo(3); + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @Test + public void testPartiallyShreddedObjectReplacement() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + assertThat(partial.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(partial.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(partial.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(partial.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(partial.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testPartiallyShreddedObjectGetMissingField() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // missing fields are returned as null + assertThat(partial.get("d")).isNull(); + } + + @Test + public void testPartiallyShreddedObjectPutMissingFieldFailure() { + ShreddedObject partial = createUnshreddedObject(FIELDS).second(); + + // d is not defined in the variant metadata and will fail + assertThatThrownBy(() -> partial.put("d", Variants.ofIsoDate("2024-10-12"))) + .isInstanceOf(IllegalArgumentException.class) + .hasMessage("Cannot find field name in metadata: d"); + } + + @Test + public void testPartiallyShreddedObjectSerializationMinimalBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject partial = pair.second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + VariantValue value = roundTripMinimalBuffer(partial, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(actual.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testPartiallyShreddedObjectSerializationLargeBuffer() { + Pair<SerializedMetadata, ShreddedObject> pair = createUnshreddedObject(FIELDS); + SerializedMetadata metadata = pair.first(); + ShreddedObject partial = pair.second(); + + // replace field c with a new value + partial.put("c", Variants.ofIsoDate("2024-10-12")); + + VariantValue value = roundTripLargeBuffer(partial, metadata); + + assertThat(value).isInstanceOf(SerializedObject.class); + SerializedObject actual = (SerializedObject) value; + + assertThat(actual.get("a")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(actual.get("b")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(actual.get("c")).isInstanceOf(VariantPrimitive.class); + assertThat(actual.get("c").type()).isEqualTo(Variants.PhysicalType.DATE); + assertThat(actual.get("c").asPrimitive().get()) + .isEqualTo(DateTimeUtil.isoDateToDays("2024-10-12")); + } + + @Test + public void testTwoByteOffsets() { + // a string larger than 255 bytes to push the value offset size above 1 byte + String randomString = RandomUtil.generateString(300, random); + SerializedPrimitive bigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("big", bigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("big").asPrimitive().get()).isEqualTo(randomString); + } + + @Test + public void testThreeByteOffsets() { + // a string larger than 65535 bytes to push the value offset size above 2 bytes + String randomString = RandomUtil.generateString(70_000, random); + SerializedPrimitive reallyBigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("really-big", reallyBigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("really-big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("really-big").asPrimitive().get()).isEqualTo(randomString); + } + + @Test + public void testFourByteOffsets() { + // a string larger than 16777215 bytes to push the value offset size above 3 bytes + String randomString = RandomUtil.generateString(16_777_300, random); + SerializedPrimitive reallyBigString = VariantTestUtil.createString(randomString); + + Map<String, VariantValue> data = Maps.newHashMap(); + data.putAll(FIELDS); + data.put("really-big", reallyBigString); + + Pair<SerializedMetadata, ShreddedObject> pair = createShreddedObject(data); + VariantValue value = roundTripLargeBuffer(pair.second(), pair.first()); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(4); + + assertThat(object.get("a").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("a").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("b").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("b").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("c").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("c").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + assertThat(object.get("really-big").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("really-big").asPrimitive().get()).isEqualTo(randomString); + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + @SuppressWarnings({"unchecked", "rawtypes"}) + public void testLargeObject(boolean sortFieldNames) { + Map<String, VariantPrimitive<String>> fields = Maps.newHashMap(); + for (int i = 0; i < 10_000; i += 1) { + fields.put( + RandomUtil.generateString(10, random), + Variants.of(RandomUtil.generateString(10, random))); + } + + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(fields.keySet(), sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, (Map) fields); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(10_000); + + for (Map.Entry<String, VariantPrimitive<String>> entry : fields.entrySet()) { + VariantValue fieldValue = object.get(entry.getKey()); + assertThat(fieldValue.type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(fieldValue.asPrimitive().get()).isEqualTo(entry.getValue().get()); + } + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testTwoByteFieldIds(boolean sortFieldNames) { + Set<String> keySet = Sets.newHashSet(); + for (int i = 0; i < 10_000; i += 1) { + keySet.add(RandomUtil.generateString(10, random)); + } + + Map<String, VariantValue> data = + ImmutableMap.of("aa", FIELDS.get("a"), "AA", FIELDS.get("b"), "ZZ", FIELDS.get("c")); + + // create metadata from the large key set and the actual keys + keySet.addAll(data.keySet()); + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(keySet, sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, data); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(3); + + assertThat(object.get("aa").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("aa").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("AA").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("AA").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("ZZ").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("ZZ").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + @ParameterizedTest + @ValueSource(booleans = {true, false}) + public void testThreeByteFieldIds(boolean sortFieldNames) { + Set<String> keySet = Sets.newHashSet(); + for (int i = 0; i < 100_000; i += 1) { + keySet.add(RandomUtil.generateString(10, random)); + } + + Map<String, VariantValue> data = + ImmutableMap.of("aa", FIELDS.get("a"), "AA", FIELDS.get("b"), "ZZ", FIELDS.get("c")); + + // create metadata from the large key set and the actual keys + keySet.addAll(data.keySet()); + SerializedMetadata metadata = + SerializedMetadata.from(VariantTestUtil.createMetadata(keySet, sortFieldNames)); + + ShreddedObject shredded = createShreddedObject(metadata, data); + VariantValue value = roundTripLargeBuffer(shredded, metadata); + + assertThat(value.type()).isEqualTo(Variants.PhysicalType.OBJECT); + SerializedObject object = (SerializedObject) value; + assertThat(object.numElements()).isEqualTo(3); + + assertThat(object.get("aa").type()).isEqualTo(Variants.PhysicalType.INT32); + assertThat(object.get("aa").asPrimitive().get()).isEqualTo(34); + assertThat(object.get("AA").type()).isEqualTo(Variants.PhysicalType.STRING); + assertThat(object.get("AA").asPrimitive().get()).isEqualTo("iceberg"); + assertThat(object.get("ZZ").type()).isEqualTo(Variants.PhysicalType.DECIMAL4); + assertThat(object.get("ZZ").asPrimitive().get()).isEqualTo(new BigDecimal("12.21")); + } + + static VariantValue roundTripMinimalBuffer(ShreddedObject object, SerializedMetadata metadata) { Review Comment: nit: private scope? -- 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: issues-unsubscr...@iceberg.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org --------------------------------------------------------------------- To unsubscribe, e-mail: issues-unsubscr...@iceberg.apache.org For additional commands, e-mail: issues-h...@iceberg.apache.org