Skip to content

Commit 7b510ad

Browse files
authored
API, AWS, Azure, Core, GCP: Use parametrized tests for Kryo/Java serialization verification (#13244)
1 parent b4f5da9 commit 7b510ad

File tree

13 files changed

+200
-402
lines changed

13 files changed

+200
-402
lines changed

api/src/test/java/org/apache/iceberg/TestHelpers.java

Lines changed: 18 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -41,17 +41,35 @@
4141
import java.util.Set;
4242
import java.util.stream.Collectors;
4343
import java.util.stream.IntStream;
44+
import java.util.stream.Stream;
4445
import org.apache.iceberg.expressions.BoundPredicate;
4546
import org.apache.iceberg.expressions.BoundSetPredicate;
4647
import org.apache.iceberg.expressions.Expression;
4748
import org.apache.iceberg.expressions.ExpressionVisitors;
4849
import org.apache.iceberg.expressions.UnboundPredicate;
4950
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
5051
import org.apache.iceberg.util.ByteBuffers;
52+
import org.junit.jupiter.api.Named;
53+
import org.junit.jupiter.params.provider.Arguments;
5154
import org.objenesis.strategy.StdInstantiatorStrategy;
5255

5356
public class TestHelpers {
5457

58+
@FunctionalInterface
59+
public interface RoundTripSerializer<T> {
60+
T apply(T obj) throws IOException, ClassNotFoundException;
61+
}
62+
63+
public static <T> Stream<Arguments> serializers() {
64+
return Stream.of(
65+
Arguments.of(
66+
Named.<RoundTripSerializer<T>>of(
67+
"KryoSerialization", TestHelpers.KryoHelpers::roundTripSerialize)),
68+
Arguments.of(
69+
Named.<RoundTripSerializer<T>>of(
70+
"JavaSerialization", TestHelpers::roundTripSerialize)));
71+
}
72+
5573
private TestHelpers() {}
5674

5775
public static final int MAX_FORMAT_VERSION = 4;

aws/src/integration/java/org/apache/iceberg/aws/s3/TestS3FileIO.java

Lines changed: 42 additions & 137 deletions
Original file line numberDiff line numberDiff line change
@@ -85,6 +85,8 @@
8585
import org.junit.jupiter.api.BeforeEach;
8686
import org.junit.jupiter.api.Disabled;
8787
import org.junit.jupiter.api.Test;
88+
import org.junit.jupiter.params.ParameterizedTest;
89+
import org.junit.jupiter.params.provider.MethodSource;
8890
import org.mockito.Mockito;
8991
import org.testcontainers.containers.MinIOContainer;
9092
import org.testcontainers.junit.jupiter.Container;
@@ -443,99 +445,53 @@ public void testFileIOJsonSerialization() {
443445
}
444446
}
445447

446-
@Test
447-
public void testS3FileIOKryoSerialization() throws IOException {
448+
@ParameterizedTest
449+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
450+
public void testS3FileIOSerialization(TestHelpers.RoundTripSerializer<FileIO> roundTripSerializer)
451+
throws IOException, ClassNotFoundException {
448452
FileIO testS3FileIO = new S3FileIO();
449453

450454
// s3 fileIO should be serializable when properties are passed as immutable map
451-
testS3FileIO.initialize(ImmutableMap.of("k1", "v1"));
452-
FileIO roundTripSerializedFileIO = TestHelpers.KryoHelpers.roundTripSerialize(testS3FileIO);
455+
testS3FileIO.initialize(ImmutableMap.of("k1", "v1", "k2", "v2"));
456+
FileIO roundTripSerializedFileIO = roundTripSerializer.apply(testS3FileIO);
453457

454458
assertThat(roundTripSerializedFileIO.properties()).isEqualTo(testS3FileIO.properties());
455459
}
456460

457-
@Test
458-
public void testS3FileIOWithEmptyPropsKryoSerialization() throws IOException {
461+
@ParameterizedTest
462+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
463+
public void testS3FileIOWithEmptyPropsSerialization(
464+
TestHelpers.RoundTripSerializer<FileIO> roundTripSerializer)
465+
throws IOException, ClassNotFoundException {
459466
FileIO testS3FileIO = new S3FileIO();
460467

461468
// s3 fileIO should be serializable when properties passed as empty immutable map
462469
testS3FileIO.initialize(ImmutableMap.of());
463-
FileIO roundTripSerializedFileIO = TestHelpers.KryoHelpers.roundTripSerialize(testS3FileIO);
470+
FileIO roundTripSerializedFileIO = roundTripSerializer.apply(testS3FileIO);
464471

465472
assertThat(roundTripSerializedFileIO.properties()).isEqualTo(testS3FileIO.properties());
466473
}
467474

468-
@Test
469-
public void fileIOWithStorageCredentialsKryoSerialization() throws IOException {
470-
S3FileIO fileIO = new S3FileIO();
471-
fileIO.setCredentials(
472-
ImmutableList.of(
473-
StorageCredential.create("prefix", Map.of("key1", "val1", "key2", "val2"))));
474-
fileIO.initialize(Map.of());
475-
476-
assertThat(TestHelpers.KryoHelpers.roundTripSerialize(fileIO).credentials())
477-
.isEqualTo(fileIO.credentials());
478-
}
479-
480-
@Test
481-
public void fileIOWithStorageCredentialsJavaSerialization()
475+
@ParameterizedTest
476+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
477+
public void fileIOWithStorageCredentialsSerialization(
478+
TestHelpers.RoundTripSerializer<S3FileIO> roundTripSerializer)
482479
throws IOException, ClassNotFoundException {
483480
S3FileIO fileIO = new S3FileIO();
484481
fileIO.setCredentials(
485482
ImmutableList.of(
486483
StorageCredential.create("prefix", Map.of("key1", "val1", "key2", "val2"))));
487484
fileIO.initialize(Map.of());
488485

489-
assertThat(TestHelpers.roundTripSerialize(fileIO).credentials())
490-
.isEqualTo(fileIO.credentials());
491-
}
492-
493-
@Test
494-
public void fileIOWithPrefixedS3ClientWithoutCredentialsKryoSerialization() throws IOException {
495-
S3FileIO io = new S3FileIO();
496-
io.initialize(Map.of(AwsClientProperties.CLIENT_REGION, "us-east-1"));
497-
498-
assertThat(io.client()).isInstanceOf(S3Client.class);
499-
assertThat(io.asyncClient()).isInstanceOf(S3AsyncClient.class);
500-
assertThat(io.client("s3a://my-bucket/my-path")).isInstanceOf(S3Client.class);
501-
assertThat(io.asyncClient("s3a://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
502-
503-
S3FileIO fileIO = TestHelpers.KryoHelpers.roundTripSerialize(io);
504-
assertThat(fileIO.credentials()).isEqualTo(io.credentials()).isEmpty();
505-
506-
assertThat(fileIO.client()).isInstanceOf(S3Client.class);
507-
assertThat(fileIO.asyncClient()).isInstanceOf(S3AsyncClient.class);
508-
assertThat(fileIO.client("s3a://my-bucket/my-path")).isInstanceOf(S3Client.class);
509-
assertThat(fileIO.asyncClient("s3a://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
486+
assertThat(roundTripSerializer.apply(fileIO).credentials()).isEqualTo(fileIO.credentials());
510487
}
511488

512-
@Test
513-
public void fileIOWithPrefixedS3ClientKryoSerialization() throws IOException {
514-
S3FileIO io = new S3FileIO();
515-
io.setCredentials(
516-
ImmutableList.of(
517-
StorageCredential.create("s3://my-bucket/my-path/table1", Map.of("key1", "val1"))));
518-
io.initialize(Map.of(AwsClientProperties.CLIENT_REGION, "us-east-1"));
519-
520-
// there should be a client for the generic and specific storage prefix available
521-
assertThat(io.client()).isInstanceOf(S3Client.class);
522-
assertThat(io.asyncClient()).isInstanceOf(S3AsyncClient.class);
523-
assertThat(io.client("s3://my-bucket/my-path")).isInstanceOf(S3Client.class);
524-
assertThat(io.asyncClient("s3://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
525-
526-
S3FileIO fileIO = TestHelpers.KryoHelpers.roundTripSerialize(io);
527-
assertThat(fileIO.credentials()).isEqualTo(io.credentials());
528-
529-
// make sure there's a client for the generic and specific storage prefix available after ser/de
530-
assertThat(fileIO.client()).isInstanceOf(S3Client.class);
531-
assertThat(fileIO.asyncClient()).isInstanceOf(S3AsyncClient.class);
532-
assertThat(fileIO.client("s3://my-bucket/my-path")).isInstanceOf(S3Client.class);
533-
assertThat(fileIO.asyncClient("s3://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
534-
}
535-
536-
@Test
537-
public void fileIOWithPrefixedS3ClientWithoutCredentialsJavaSerialization()
489+
@ParameterizedTest
490+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
491+
public void fileIOWithPrefixedS3ClientWithoutCredentialsSerialization(
492+
TestHelpers.RoundTripSerializer<S3FileIO> roundTripSerializer)
538493
throws IOException, ClassNotFoundException {
494+
539495
S3FileIO io = new S3FileIO();
540496
io.initialize(Map.of(AwsClientProperties.CLIENT_REGION, "us-east-1"));
541497

@@ -544,7 +500,7 @@ public void fileIOWithPrefixedS3ClientWithoutCredentialsJavaSerialization()
544500
assertThat(io.client("s3a://my-bucket/my-path")).isInstanceOf(S3Client.class);
545501
assertThat(io.asyncClient("s3a://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
546502

547-
S3FileIO fileIO = TestHelpers.roundTripSerialize(io);
503+
S3FileIO fileIO = roundTripSerializer.apply(io);
548504
assertThat(fileIO.credentials()).isEqualTo(io.credentials()).isEmpty();
549505

550506
assertThat(fileIO.client()).isInstanceOf(S3Client.class);
@@ -553,8 +509,10 @@ public void fileIOWithPrefixedS3ClientWithoutCredentialsJavaSerialization()
553509
assertThat(fileIO.asyncClient("s3a://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
554510
}
555511

556-
@Test
557-
public void fileIOWithPrefixedS3ClientJavaSerialization()
512+
@ParameterizedTest
513+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
514+
public void fileIOWithPrefixedS3ClientSerialization(
515+
TestHelpers.RoundTripSerializer<S3FileIO> roundTripSerializer)
558516
throws IOException, ClassNotFoundException {
559517
S3FileIO io = new S3FileIO();
560518
io.setCredentials(
@@ -568,7 +526,7 @@ public void fileIOWithPrefixedS3ClientJavaSerialization()
568526
assertThat(io.client("s3://my-bucket/my-path")).isInstanceOf(S3Client.class);
569527
assertThat(io.asyncClient("s3://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
570528

571-
S3FileIO fileIO = TestHelpers.roundTripSerialize(io);
529+
S3FileIO fileIO = roundTripSerializer.apply(io);
572530
assertThat(fileIO.credentials()).isEqualTo(io.credentials());
573531

574532
// make sure there's a client for the generic and specific storage prefix available after ser/de
@@ -578,17 +536,6 @@ public void fileIOWithPrefixedS3ClientJavaSerialization()
578536
assertThat(fileIO.asyncClient("s3://my-bucket/my-path")).isInstanceOf(S3AsyncClient.class);
579537
}
580538

581-
@Test
582-
public void testS3FileIOJavaSerialization() throws IOException, ClassNotFoundException {
583-
FileIO testS3FileIO = new S3FileIO();
584-
585-
// s3 fileIO should be serializable when properties are passed as immutable map
586-
testS3FileIO.initialize(ImmutableMap.of("k1", "v1"));
587-
FileIO roundTripSerializedFileIO = TestHelpers.roundTripSerialize(testS3FileIO);
588-
589-
assertThat(roundTripSerializedFileIO.properties()).isEqualTo(testS3FileIO.properties());
590-
}
591-
592539
@Test
593540
public void testResolvingFileIOLoad() {
594541
ResolvingFileIO resolvingFileIO = new ResolvingFileIO();
@@ -662,8 +609,10 @@ public void testInputFileWithManifest() throws IOException {
662609
verify(s3mock, never()).headObject(any(HeadObjectRequest.class));
663610
}
664611

665-
@Test
666-
public void resolvingFileIOLoadWithoutStorageCredentials()
612+
@ParameterizedTest
613+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
614+
public void resolvingFileIOLoadWithoutStorageCredentials(
615+
TestHelpers.RoundTripSerializer<ResolvingFileIO> roundTripSerializer)
667616
throws IOException, ClassNotFoundException {
668617
ResolvingFileIO resolvingFileIO = new ResolvingFileIO();
669618
resolvingFileIO.initialize(ImmutableMap.of(AwsClientProperties.CLIENT_REGION, "us-east-1"));
@@ -686,8 +635,8 @@ public void resolvingFileIOLoadWithoutStorageCredentials()
686635
.isInstanceOf(S3AsyncClient.class);
687636
});
688637

689-
// make sure credentials can be accessed after kryo serde
690-
ResolvingFileIO resolvingIO = TestHelpers.KryoHelpers.roundTripSerialize(resolvingFileIO);
638+
// make sure credentials can be accessed after serde
639+
ResolvingFileIO resolvingIO = roundTripSerializer.apply(resolvingFileIO);
691640
assertThat(resolvingIO.credentials()).isEmpty();
692641
result =
693642
DynMethods.builder("io")
@@ -706,31 +655,12 @@ public void resolvingFileIOLoadWithoutStorageCredentials()
706655
.isSameAs(fileIO.asyncClient())
707656
.isInstanceOf(S3AsyncClient.class);
708657
});
709-
710-
// make sure credentials can be accessed after java serde
711-
resolvingIO = TestHelpers.roundTripSerialize(resolvingFileIO);
712-
assertThat(resolvingIO.credentials()).isEmpty();
713-
result =
714-
DynMethods.builder("io")
715-
.hiddenImpl(ResolvingFileIO.class, String.class)
716-
.build(resolvingIO)
717-
.invoke("s3://foo/bar");
718-
assertThat(result)
719-
.isInstanceOf(S3FileIO.class)
720-
.asInstanceOf(InstanceOfAssertFactories.type(S3FileIO.class))
721-
.satisfies(
722-
fileIO -> {
723-
assertThat(fileIO.client("s3://foo/bar"))
724-
.isSameAs(fileIO.client())
725-
.isInstanceOf(S3Client.class);
726-
assertThat(fileIO.asyncClient("s3://foo/bar"))
727-
.isSameAs(fileIO.asyncClient())
728-
.isInstanceOf(S3AsyncClient.class);
729-
});
730658
}
731659

732-
@Test
733-
public void resolvingFileIOLoadWithStorageCredentials()
660+
@ParameterizedTest
661+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
662+
public void resolvingFileIOLoadWithStorageCredentials(
663+
TestHelpers.RoundTripSerializer<ResolvingFileIO> roundTripSerializer)
734664
throws IOException, ClassNotFoundException {
735665
StorageCredential credential = StorageCredential.create("s3://foo/bar", Map.of("key1", "val1"));
736666
List<StorageCredential> storageCredentials = ImmutableList.of(credential);
@@ -760,33 +690,8 @@ public void resolvingFileIOLoadWithStorageCredentials()
760690
.isInstanceOf(S3AsyncClient.class);
761691
});
762692

763-
// make sure credentials are still present after kryo serde
764-
ResolvingFileIO resolvingIO = TestHelpers.KryoHelpers.roundTripSerialize(resolvingFileIO);
765-
assertThat(resolvingIO.credentials()).isEqualTo(storageCredentials);
766-
result =
767-
DynMethods.builder("io")
768-
.hiddenImpl(ResolvingFileIO.class, String.class)
769-
.build(resolvingIO)
770-
.invoke("s3://foo/bar");
771-
io =
772-
assertThat(result)
773-
.isInstanceOf(S3FileIO.class)
774-
.asInstanceOf(InstanceOfAssertFactories.type(S3FileIO.class));
775-
io.extracting(S3FileIO::credentials).isEqualTo(storageCredentials);
776-
io.satisfies(
777-
fileIO -> {
778-
// make sure there are two separate S3 clients for different prefixes and that the
779-
// underlying sync/async client is set
780-
assertThat(fileIO.client("s3://foo/bar"))
781-
.isNotSameAs(fileIO.client())
782-
.isInstanceOf(S3Client.class);
783-
assertThat(fileIO.asyncClient("s3://foo/bar"))
784-
.isNotSameAs(fileIO.asyncClient())
785-
.isInstanceOf(S3AsyncClient.class);
786-
});
787-
788-
// make sure credentials are still present after java serde
789-
resolvingIO = TestHelpers.roundTripSerialize(resolvingFileIO);
693+
// make sure credentials are still present after serde
694+
ResolvingFileIO resolvingIO = roundTripSerializer.apply(resolvingFileIO);
790695
assertThat(resolvingIO.credentials()).isEqualTo(storageCredentials);
791696
result =
792697
DynMethods.builder("io")

aws/src/test/java/org/apache/iceberg/aws/TestAwsClientFactories.java

Lines changed: 20 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -33,6 +33,8 @@
3333
import org.apache.iceberg.util.SerializationUtil;
3434
import org.assertj.core.api.ThrowableAssert;
3535
import org.junit.jupiter.api.Test;
36+
import org.junit.jupiter.params.ParameterizedTest;
37+
import org.junit.jupiter.params.provider.MethodSource;
3638
import software.amazon.awssdk.auth.credentials.AwsBasicCredentials;
3739
import software.amazon.awssdk.auth.credentials.AwsCredentials;
3840
import software.amazon.awssdk.auth.credentials.AwsCredentialsProvider;
@@ -133,12 +135,14 @@ public void testS3FileIoCredentialsVerification() {
133135
.hasMessage("S3 client access key ID and secret access key must be set at the same time");
134136
}
135137

136-
@Test
137-
public void testDefaultAwsClientFactorySerializable() throws IOException {
138+
@ParameterizedTest
139+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
140+
public void testDefaultAwsClientFactorySerializable(
141+
TestHelpers.RoundTripSerializer<AwsClientFactory> roundTripSerializer)
142+
throws IOException, ClassNotFoundException {
138143
Map<String, String> properties = Maps.newHashMap();
139144
AwsClientFactory defaultAwsClientFactory = AwsClientFactories.from(properties);
140-
AwsClientFactory roundTripResult =
141-
TestHelpers.KryoHelpers.roundTripSerialize(defaultAwsClientFactory);
145+
AwsClientFactory roundTripResult = roundTripSerializer.apply(defaultAwsClientFactory);
142146
assertThat(roundTripResult).isInstanceOf(AwsClientFactories.DefaultAwsClientFactory.class);
143147

144148
byte[] serializedFactoryBytes = SerializationUtil.serializeToBytes(defaultAwsClientFactory);
@@ -148,15 +152,17 @@ public void testDefaultAwsClientFactorySerializable() throws IOException {
148152
.isInstanceOf(AwsClientFactories.DefaultAwsClientFactory.class);
149153
}
150154

151-
@Test
152-
public void testAssumeRoleAwsClientFactorySerializable() throws IOException {
155+
@ParameterizedTest
156+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
157+
public void testAssumeRoleAwsClientFactorySerializable(
158+
TestHelpers.RoundTripSerializer<AwsClientFactory> roundTripSerializer)
159+
throws IOException, ClassNotFoundException {
153160
Map<String, String> properties = Maps.newHashMap();
154161
properties.put(AwsProperties.CLIENT_FACTORY, AssumeRoleAwsClientFactory.class.getName());
155162
properties.put(AwsProperties.CLIENT_ASSUME_ROLE_ARN, "arn::test");
156163
properties.put(AwsProperties.CLIENT_ASSUME_ROLE_REGION, "us-east-1");
157164
AwsClientFactory assumeRoleAwsClientFactory = AwsClientFactories.from(properties);
158-
AwsClientFactory roundTripResult =
159-
TestHelpers.KryoHelpers.roundTripSerialize(assumeRoleAwsClientFactory);
165+
AwsClientFactory roundTripResult = roundTripSerializer.apply(assumeRoleAwsClientFactory);
160166
assertThat(roundTripResult).isInstanceOf(AssumeRoleAwsClientFactory.class);
161167

162168
byte[] serializedFactoryBytes = SerializationUtil.serializeToBytes(assumeRoleAwsClientFactory);
@@ -165,8 +171,11 @@ public void testAssumeRoleAwsClientFactorySerializable() throws IOException {
165171
assertThat(deserializedClientFactory).isInstanceOf(AssumeRoleAwsClientFactory.class);
166172
}
167173

168-
@Test
169-
public void testLakeFormationAwsClientFactorySerializable() throws IOException {
174+
@ParameterizedTest
175+
@MethodSource("org.apache.iceberg.TestHelpers#serializers")
176+
public void testLakeFormationAwsClientFactorySerializable(
177+
TestHelpers.RoundTripSerializer<AwsClientFactory> roundTripSerializer)
178+
throws IOException, ClassNotFoundException {
170179
Map<String, String> properties = Maps.newHashMap();
171180
properties.put(AwsProperties.CLIENT_FACTORY, LakeFormationAwsClientFactory.class.getName());
172181
properties.put(AwsProperties.CLIENT_ASSUME_ROLE_ARN, "arn::test");
@@ -176,8 +185,7 @@ public void testLakeFormationAwsClientFactorySerializable() throws IOException {
176185
+ LakeFormationAwsClientFactory.LF_AUTHORIZED_CALLER,
177186
"emr");
178187
AwsClientFactory lakeFormationAwsClientFactory = AwsClientFactories.from(properties);
179-
AwsClientFactory roundTripResult =
180-
TestHelpers.KryoHelpers.roundTripSerialize(lakeFormationAwsClientFactory);
188+
AwsClientFactory roundTripResult = roundTripSerializer.apply(lakeFormationAwsClientFactory);
181189
assertThat(roundTripResult).isInstanceOf(LakeFormationAwsClientFactory.class);
182190

183191
byte[] serializedFactoryBytes =

0 commit comments

Comments
 (0)