提交 f81af459 编写于 作者: S Stephan Ewen

[hotfix] [tests] Move Kryo Registrations test to flink-runtime

上级 991af365
/*
* 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.flink.runtime.misc;
import org.apache.flink.api.common.ExecutionConfig;
import org.apache.flink.api.java.typeutils.runtime.kryo.KryoSerializer;
import com.esotericsoftware.kryo.Kryo;
import com.esotericsoftware.kryo.Registration;
import org.junit.Test;
import java.io.BufferedReader;
import java.io.BufferedWriter;
import java.io.File;
import java.io.FileWriter;
import java.io.IOException;
import java.io.InputStreamReader;
import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail;
/**
* Tests that the set of Kryo registrations is the same across compatible
* Flink versions.
*
* <p>The test needs to be in the runtime package, rather than in the
* core/java package where the Kryo serializer itself sits, because
* when runtime is present in the classpath, Chill is used to instantiate
* Kryo and adds the proper set of registrations.
*/
public class KryoSerializerRegistrationsTest {
/**
* Tests that the registered classes in Kryo did not change.
*
* <p>Once we have proper serializer versioning this test will become obsolete.
* But currently a change in the serializers can break savepoint backwards
* compatibility between Flink versions.
*/
@Test
public void testDefaultKryoRegisteredClassesDidNotChange() throws Exception {
final Kryo kryo = new KryoSerializer<>(Integer.class, new ExecutionConfig()).getKryo();
try (BufferedReader reader = new BufferedReader(new InputStreamReader(
getClass().getClassLoader().getResourceAsStream("flink_11-kryo_registrations")))) {
String line;
while ((line = reader.readLine()) != null) {
String[] split = line.split(",");
final int tag = Integer.parseInt(split[0]);
final String registeredClass = split[1];
Registration registration = kryo.getRegistration(tag);
if (registration == null) {
fail(String.format("Registration for %d = %s got lost", tag, registeredClass));
}
else if (!registeredClass.equals(registration.getType().getName())) {
fail(String.format("Registration for %d = %s changed to %s",
tag, registeredClass, registration.getType().getName()));
}
}
}
}
/**
* Creates a Kryo serializer and writes the default registrations out to a
* comma separated file with one entry per line:
*
* <pre>
* id,class
* </pre>
*
* <p>The produced file is used to check that the registered IDs don't change
* in future Flink versions.
*
* <p>This method is not used in the tests, but documents how the test file
* has been created and can be used to re-create it if needed.
*
* @param filePath File path to write registrations to
*/
private void writeDefaultKryoRegistrations(String filePath) throws IOException {
final File file = new File(filePath);
if (file.exists()) {
assertTrue(file.delete());
}
final Kryo kryo = new KryoSerializer<>(Integer.class, new ExecutionConfig()).getKryo();
final int nextId = kryo.getNextRegistrationId();
try (BufferedWriter writer = new BufferedWriter(new FileWriter(file))) {
for (int i = 0; i < nextId; i++) {
Registration registration = kryo.getRegistration(i);
String str = registration.getId() + "," + registration.getType().getName();
writer.write(str, 0, str.length());
writer.newLine();
}
System.out.println("Created file with registrations at " + file.getAbsolutePath());
}
}
}
0,int
1,java.lang.String
2,float
3,boolean
4,byte
5,char
6,short
7,long
8,double
9,void
10,scala.collection.convert.Wrappers$SeqWrapper
11,scala.collection.convert.Wrappers$IteratorWrapper
12,scala.collection.convert.Wrappers$MapWrapper
13,scala.collection.convert.Wrappers$JListWrapper
14,scala.collection.convert.Wrappers$JMapWrapper
15,scala.Some
16,scala.util.Left
17,scala.util.Right
18,scala.collection.immutable.Vector
19,scala.collection.immutable.Set$Set1
20,scala.collection.immutable.Set$Set2
21,scala.collection.immutable.Set$Set3
22,scala.collection.immutable.Set$Set4
23,scala.collection.immutable.HashSet$HashTrieSet
24,scala.collection.immutable.Map$Map1
25,scala.collection.immutable.Map$Map2
26,scala.collection.immutable.Map$Map3
27,scala.collection.immutable.Map$Map4
28,scala.collection.immutable.HashMap$HashTrieMap
29,scala.collection.immutable.Range$Inclusive
30,scala.collection.immutable.NumericRange$Inclusive
31,scala.collection.immutable.NumericRange$Exclusive
32,scala.collection.mutable.BitSet
33,scala.collection.mutable.HashMap
34,scala.collection.mutable.HashSet
35,scala.collection.convert.Wrappers$IterableWrapper
36,scala.Tuple1
37,scala.Tuple2
38,scala.Tuple3
39,scala.Tuple4
40,scala.Tuple5
41,scala.Tuple6
42,scala.Tuple7
43,scala.Tuple8
44,scala.Tuple9
45,scala.Tuple10
46,scala.Tuple11
47,scala.Tuple12
48,scala.Tuple13
49,scala.Tuple14
50,scala.Tuple15
51,scala.Tuple16
52,scala.Tuple17
53,scala.Tuple18
54,scala.Tuple19
55,scala.Tuple20
56,scala.Tuple21
57,scala.Tuple22
58,scala.Tuple1$mcJ$sp
59,scala.Tuple1$mcI$sp
60,scala.Tuple1$mcD$sp
61,scala.Tuple2$mcJJ$sp
62,scala.Tuple2$mcJI$sp
63,scala.Tuple2$mcJD$sp
64,scala.Tuple2$mcIJ$sp
65,scala.Tuple2$mcII$sp
66,scala.Tuple2$mcID$sp
67,scala.Tuple2$mcDJ$sp
68,scala.Tuple2$mcDI$sp
69,scala.Tuple2$mcDD$sp
70,scala.Symbol
71,scala.reflect.ClassTag
72,scala.runtime.BoxedUnit
73,java.util.Arrays$ArrayList
74,java.util.BitSet
75,java.util.PriorityQueue
76,java.util.regex.Pattern
77,java.sql.Date
78,java.sql.Time
79,java.sql.Timestamp
80,java.net.URI
81,java.net.InetSocketAddress
82,java.util.UUID
83,java.util.Locale
84,java.text.SimpleDateFormat
85,org.apache.avro.generic.GenericData$Array
......@@ -1044,7 +1044,7 @@ under the License.
<!-- Test Data. -->
<exclude>flink-tests/src/test/resources/testdata/terainput.txt</exclude>
<exclude>flink-tests/src/test/resources/flink_11-kryo_registrations</exclude>
<exclude>flink-runtime/src/test/resources/flink_11-kryo_registrations</exclude>
<exclude>flink-connectors/flink-avro/src/test/resources/avro/*.avsc</exclude>
<exclude>out/test/flink-avro/avro/user.avsc</exclude>
<exclude>flink-libraries/flink-table/src/test/scala/resources/*.out</exclude>
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册