You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@ignite.apache.org by GitBox <gi...@apache.org> on 2021/12/08 13:02:02 UTC

[GitHub] [ignite-3] rpuch commented on a change in pull request #495: IGNITE-15940 Class descriptor parser

rpuch commented on a change in pull request #495:
URL: https://github.com/apache/ignite-3/pull/495#discussion_r764753510



##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/SerializationType.java
##########
@@ -0,0 +1,58 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import org.apache.ignite.internal.network.serialization.DefaultClassDescriptors.DefaultType;
+
+/**
+ * Serialization type.
+ */
+public enum SerializationType {
+    /** Used for predefined descriptors like primitive (or boxed int). See {@link DefaultType}. */
+    DEFAULT,
+
+    /** Type for not serializable or externalizable classes.  */

Review comment:
       Do you mean 'for classes that are neither serializable nor externalizable' or 'something that is not serializable, but count be externalizable'?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/FieldDescriptor.java
##########
@@ -0,0 +1,80 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import org.jetbrains.annotations.NotNull;
+
+/**
+ * Field descriptor for the user object serialization.
+ */
+public class FieldDescriptor {
+    /**
+     * Name of the field.
+     */
+    @NotNull
+    private final String name;
+
+    /**
+     * Type of the field.
+     */
+    @NotNull
+    private final Class<?> clazz;
+
+    /**
+     * Type descriptor of the field.
+     */
+    private final int typeDescriptor;

Review comment:
       Would `typeDescriptorId` be more accurate? As it's not the descriptor itself, just its ID.

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),
+        STRING(18, String.class),
+        UUID(19, UUID.class),
+        IGNITE_UUID(20, IgniteUuid.class),
+        DATE(21, Date.class),

Review comment:
       Aren't we planning to support java.time.* classes like `Instant`, `LocalTime` and so on?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParserContext.java
##########
@@ -0,0 +1,126 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.util.Collection;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.atomic.AtomicInteger;
+import org.apache.ignite.internal.network.serialization.DefaultClassDescriptors.DefaultType;
+import org.jetbrains.annotations.Nullable;
+
+/**
+ * Class parser context.
+ */
+public class ClassDescriptorParserContext {
+    /** Sequential id generator for class descriptors. */
+    private final AtomicInteger idGenerator = new AtomicInteger(100);
+
+    /** Map class -> descriptor id. */
+    private final ConcurrentMap<Class<?>, Integer> idMap = new ConcurrentHashMap<>();
+
+    /** Map descriptor id -> class descriptor. */
+    private final ConcurrentMap<Integer, ClassDescriptor> descriptorMap = new ConcurrentHashMap<>();
+
+    /**
+     * Constructor.
+     */
+    public ClassDescriptorParserContext() {
+        for (DefaultType value : DefaultType.values()) {
+            ClassDescriptor defaultDescriptor = DefaultClassDescriptors.createDefaultDescriptor(
+                    value);
+
+            addPredefinedDescriptor(value.clazz(), defaultDescriptor);
+        }
+    }
+
+    /**
+     * Adds predefined class descriptor with a statically configured id.
+     *
+     * @param clazz Class.
+     * @param descriptor Descriptor.
+     */
+    private void addPredefinedDescriptor(Class<?> clazz, ClassDescriptor descriptor) {
+        int descriptorId = descriptor.descriptorId();
+
+        Integer existingId = idMap.put(clazz, descriptorId);
+
+        assert existingId == null;
+
+        ClassDescriptor existingDescriptor = descriptorMap.put(descriptorId, descriptor);
+
+        assert existingDescriptor == null;
+    }
+
+    /**
+     * Gets descriptor id for the class.
+     *
+     * @param clazz Class.
+     * @return Descriptor id.
+     */
+    public int getId(Class<?> clazz) {
+        return idMap.computeIfAbsent(clazz, unused -> idGenerator.getAndIncrement());
+    }
+
+    /**
+     * Gets a descriptor by the id.
+     *
+     * @param descriptorId Descriptor id.
+     * @return Descriptor.
+     */
+    @Nullable
+    public ClassDescriptor getDescriptor(int descriptorId) {
+        return descriptorMap.get(descriptorId);
+    }
+
+    /**
+     * Gets a descriptor by the class.
+     *
+     * @param clazz Class.
+     * @return Descriptor.
+     */
+    @Nullable
+    public ClassDescriptor getDescriptor(Class<?> clazz) {
+        Integer descriptorId = idMap.get(clazz);
+
+        if (descriptorId == null) {
+            return null;
+        }
+
+        return descriptorMap.get(descriptorId);
+    }
+
+    /**
+     * Returns {@code true} if there is a descriptor for the id.
+     *
+     * @param descriptorId Descriptor id.
+     * @return {@code true} if there is a descriptor for the id.
+     */
+    public boolean hasDescriptor(int descriptorId) {
+        return descriptorMap.containsKey(descriptorId);
+    }
+
+    /**
+     * Adds a list of descriptors.
+     *
+     * @param descriptors Descriptors.
+     */
+    public void addDescriptors(Collection<ClassDescriptor> descriptors) {
+        descriptors.forEach(descriptor -> descriptorMap.put(descriptor.descriptorId(), descriptor));

Review comment:
       Is it ok that `idMap` is not updated here? Not even checked for consistency?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParserContext.java
##########
@@ -0,0 +1,126 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.util.Collection;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.atomic.AtomicInteger;
+import org.apache.ignite.internal.network.serialization.DefaultClassDescriptors.DefaultType;
+import org.jetbrains.annotations.Nullable;
+
+/**
+ * Class parser context.
+ */
+public class ClassDescriptorParserContext {
+    /** Sequential id generator for class descriptors. */
+    private final AtomicInteger idGenerator = new AtomicInteger(100);
+
+    /** Map class -> descriptor id. */
+    private final ConcurrentMap<Class<?>, Integer> idMap = new ConcurrentHashMap<>();
+
+    /** Map descriptor id -> class descriptor. */
+    private final ConcurrentMap<Integer, ClassDescriptor> descriptorMap = new ConcurrentHashMap<>();
+
+    /**
+     * Constructor.
+     */
+    public ClassDescriptorParserContext() {
+        for (DefaultType value : DefaultType.values()) {
+            ClassDescriptor defaultDescriptor = DefaultClassDescriptors.createDefaultDescriptor(
+                    value);
+
+            addPredefinedDescriptor(value.clazz(), defaultDescriptor);
+        }
+    }
+
+    /**
+     * Adds predefined class descriptor with a statically configured id.
+     *
+     * @param clazz Class.
+     * @param descriptor Descriptor.
+     */
+    private void addPredefinedDescriptor(Class<?> clazz, ClassDescriptor descriptor) {
+        int descriptorId = descriptor.descriptorId();
+
+        Integer existingId = idMap.put(clazz, descriptorId);
+
+        assert existingId == null;

Review comment:
       Same thing about assertions here and all over the PR :)

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);

Review comment:
       These two `Method` objects are only used for comparison with `null`. I suggest doing something like this:
   
   ```
   boolean hasWriteReplace = hasWriteReplace(clazz);
   boolean hasReadResolve = hasReadResolve(clazz);
   ```
   
   and then use the booleans in the `if` conditions. This would allow to make the conditions a bit more readable (less `==` and `!=` noise).

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);
+        Method readResolve = getReadResolve(clazz);
+
+        SerializationType type;
+
+        if (overrideSerialization) {

Review comment:
       Would it read in a more obvious way with a 3-level nested `if` structure? Like
   
   ```
   if (overrideSerialization) {
       if (writeReplace) {
           if (readResolve) {
               ...
           } else {
               ...
   ```

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),
+        STRING(18, String.class),
+        UUID(19, UUID.class),
+        IGNITE_UUID(20, IgniteUuid.class),
+        DATE(21, Date.class),
+        BYTE_ARRAY(22, byte[].class),
+        SHORT_ARRAY(23, short[].class),
+        INT_ARRAY(24, int[].class),
+        FLOAT_ARRAY(25, float[].class),
+        LONG_ARRAY(26, long[].class),
+        DOUBLE_ARRAY(27, double[].class),
+        BOOLEAN_ARRAY(28, boolean[].class),
+        CHAR_ARRAY(29, char[].class),
+        OBJECT_ARRAY(30, Object[].class),
+        STRING_ARRAY(31, String[].class),
+        DECIMAL(32, BigDecimal.class),
+        DECIMAL_ARRAY(33, BigDecimal[].class),
+        ENUM(34, Enum.class),
+        ENUM_ARRAY(35, Enum[].class),
+        ARRAY_LIST(36, ArrayList.class),
+        LINKED_LIST(37, LinkedList.class),
+        HASH_SET(38, HashSet.class),
+        LINKED_HASH_SET(39, LinkedHashSet.class),
+        SINGLETON_LIST(40, Collections.singletonList(null).getClass()),
+        HASH_MAP(41, HashMap.class),
+        LINKED_HASH_MAP(42, LinkedHashMap.class),
+        BIT_SET(43, BitSet.class),
+        NULL(44, Null.class),

Review comment:
       When could be `NULL` used? Why not use `OBJECT` for the cases when we don't know the actual reference type?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),

Review comment:
       I suggest adding a dumb test that would test these IDs, like `assertThat(DefaultType.FLOAT, is(7))` just to make sure that a cat running over someones keyboard does not introduce us a bug that would be hard to spot.

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),
+        STRING(18, String.class),
+        UUID(19, UUID.class),
+        IGNITE_UUID(20, IgniteUuid.class),
+        DATE(21, Date.class),
+        BYTE_ARRAY(22, byte[].class),
+        SHORT_ARRAY(23, short[].class),
+        INT_ARRAY(24, int[].class),
+        FLOAT_ARRAY(25, float[].class),
+        LONG_ARRAY(26, long[].class),
+        DOUBLE_ARRAY(27, double[].class),
+        BOOLEAN_ARRAY(28, boolean[].class),
+        CHAR_ARRAY(29, char[].class),
+        OBJECT_ARRAY(30, Object[].class),
+        STRING_ARRAY(31, String[].class),
+        DECIMAL(32, BigDecimal.class),
+        DECIMAL_ARRAY(33, BigDecimal[].class),
+        ENUM(34, Enum.class),
+        ENUM_ARRAY(35, Enum[].class),
+        ARRAY_LIST(36, ArrayList.class),
+        LINKED_LIST(37, LinkedList.class),
+        HASH_SET(38, HashSet.class),
+        LINKED_HASH_SET(39, LinkedHashSet.class),
+        SINGLETON_LIST(40, Collections.singletonList(null).getClass()),

Review comment:
       The class implementing `singletonList()` is a Java standard library imlementation detail, they could change it at any time. Do we want to depend on such internal details?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),
+        STRING(18, String.class),
+        UUID(19, UUID.class),
+        IGNITE_UUID(20, IgniteUuid.class),
+        DATE(21, Date.class),
+        BYTE_ARRAY(22, byte[].class),
+        SHORT_ARRAY(23, short[].class),
+        INT_ARRAY(24, int[].class),
+        FLOAT_ARRAY(25, float[].class),
+        LONG_ARRAY(26, long[].class),
+        DOUBLE_ARRAY(27, double[].class),
+        BOOLEAN_ARRAY(28, boolean[].class),
+        CHAR_ARRAY(29, char[].class),
+        OBJECT_ARRAY(30, Object[].class),
+        STRING_ARRAY(31, String[].class),

Review comment:
       Why are `String` array, `Decimal` array and so on treated in a special way, not just as `Object` arrays?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),

Review comment:
       How about `PRIMITIVE` instead of `P` (as `P` makes you wonder what it means)?
   
   Also, this can be called the other way around: `BYTE` for primitive and `BYTE_WRAPPER` (or something similar) for the wrapper type. Is there any good reason to present wrapper types 'more default' than the primitive ones?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),

Review comment:
       15 seems to be missing, is this ok?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParserContext.java
##########
@@ -0,0 +1,126 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.util.Collection;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.atomic.AtomicInteger;
+import org.apache.ignite.internal.network.serialization.DefaultClassDescriptors.DefaultType;
+import org.jetbrains.annotations.Nullable;
+
+/**
+ * Class parser context.
+ */
+public class ClassDescriptorParserContext {
+    /** Sequential id generator for class descriptors. */
+    private final AtomicInteger idGenerator = new AtomicInteger(100);

Review comment:
       Is 100 enough for built-in descriptors? :)

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :

Review comment:
       How about using something that works in production code to make this assertion (like `A.ensure()` in Ignite 2)? It's a bit scary to remove such protections when they are needed most :) I doubt that class descriptors will be parsed a million times per second.

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");

Review comment:
       !

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/DefaultClassDescriptors.java
##########
@@ -0,0 +1,140 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.math.BigDecimal;
+import java.util.ArrayList;
+import java.util.BitSet;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashMap;
+import java.util.LinkedHashSet;
+import java.util.LinkedList;
+import java.util.UUID;
+import org.apache.ignite.lang.IgniteUuid;
+
+/**
+ * Class that holds default types and creates descriptors for them.
+ */
+public class DefaultClassDescriptors {
+    /**
+     * Creates a descriptor for one of the default types.
+     *
+     * @param type Default type.
+     * @return Descriptor.
+     */
+    public static ClassDescriptor createDefaultDescriptor(DefaultType type) {
+        Class<?> clazz = type.clazz;
+        int descriptorId = type.descriptorId;
+        return new ClassDescriptor(
+            ClassDescriptorParser.className(clazz),
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.DEFAULT,
+            ClassDescriptorParser.isFinal(clazz)
+        );
+    }
+
+    /**
+     * Default types.
+     */
+    public enum DefaultType {
+        BYTE_P(0, byte.class),
+        BYTE(1, Byte.class),
+        SHORT_P(2, short.class),
+        SHORT(3, Short.class),
+        INT_P(4, int.class),
+        INT(5, Integer.class),
+        FLOAT_P(6, float.class),
+        FLOAT(7, Float.class),
+        LONG_P(8, long.class),
+        LONG(9, Long.class),
+        DOUBLE_P(10, double.class),
+        DOUBLE(11, Double.class),
+        BOOLEAN_P(12, boolean.class),
+        BOOLEAN(13, Boolean.class),
+        CHAR_P(14, char.class),
+        CHAR(16, Character.class),
+        OBJECT(17, Object.class),
+        STRING(18, String.class),
+        UUID(19, UUID.class),
+        IGNITE_UUID(20, IgniteUuid.class),
+        DATE(21, Date.class),
+        BYTE_ARRAY(22, byte[].class),
+        SHORT_ARRAY(23, short[].class),
+        INT_ARRAY(24, int[].class),
+        FLOAT_ARRAY(25, float[].class),
+        LONG_ARRAY(26, long[].class),
+        DOUBLE_ARRAY(27, double[].class),
+        BOOLEAN_ARRAY(28, boolean[].class),
+        CHAR_ARRAY(29, char[].class),
+        OBJECT_ARRAY(30, Object[].class),
+        STRING_ARRAY(31, String[].class),
+        DECIMAL(32, BigDecimal.class),
+        DECIMAL_ARRAY(33, BigDecimal[].class),
+        ENUM(34, Enum.class),
+        ENUM_ARRAY(35, Enum[].class),
+        ARRAY_LIST(36, ArrayList.class),

Review comment:
       Why are all these specific implementations given their own constants here? Wouldn't just `List` be enough?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {

Review comment:
       I suggest to extract this code to a method with a name like `checkHasPublicNoArgConstructor()`, it will hide the gory details and help make the `externalizable()` code clearer.

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {

Review comment:
       How about extracting a method `overridesSerialization()` returning `boolean`?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptor.java
##########
@@ -0,0 +1,116 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.util.List;
+import org.jetbrains.annotations.NotNull;
+
+/**
+ * Class descriptor for the user object serialization.
+ */
+public class ClassDescriptor {
+
+    /**
+     * Name of the class.
+     */
+    @NotNull
+    private final String className;
+
+    /**
+     * Descriptor id.
+     */
+    private final int descriptorId;
+
+    /**
+     * List of the classes fields' descriptors.
+     */
+    @NotNull
+    private final List<FieldDescriptor> fields;
+
+    /**
+     * The type of the serialization mechanism for the class.
+     */
+    @NotNull
+    private final SerializationType serializationType;
+
+    /**
+     * Whether the class is final.
+     */
+    private final boolean isFinal;
+
+    /**
+     * Constructor.
+     */
+    public ClassDescriptor(@NotNull String className, int descriptorId,
+            @NotNull List<FieldDescriptor> fields, @NotNull SerializationType serializationType,
+            boolean isFinal) {
+        this.className = className;
+        this.descriptorId = descriptorId;
+        this.fields = fields;

Review comment:
       Shouldn't we make a defensive copy here? Just in case.

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);
+        Method readResolve = getReadResolve(clazz);
+
+        SerializationType type;
+
+        if (overrideSerialization) {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        } else {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        }
+
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            type,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the arbitrary class (not serializable or externalizable) definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Arbitrary class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor arbitrary(int descriptorId, Class<?> clazz) {
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            SerializationType.ARBITRARY,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Gets field descriptors of the class. If a field's type doesn't have an id yet, generates it.
+     *
+     * @param clazz Class.
+     * @return List of field descriptor.
+     */
+    private List<FieldDescriptor> fields(Class<?> clazz) {
+        if (clazz.getSuperclass() != Object.class) {
+            // TODO: IGNITE-15945 add support for the inheritance
+            throw new UnsupportedOperationException("IGNITE-15945");
+        }
+
+        Field[] fields = clazz.getDeclaredFields();
+
+        List<FieldDescriptor> descs = new ArrayList<>(fields.length);
+
+        for (Field field : fields) {
+            Class<?> type = field.getType();
+
+            int modifiers = field.getModifiers();
+
+            // Ignore static and transient field.
+            if (Modifier.isStatic(modifiers) || Modifier.isTransient(modifiers)) {
+                continue;
+            }
+
+            String name = field.getName();
+
+            // Get or create type id.
+            int typeId = context.getId(type);
+
+            FieldDescriptor fieldDescriptor = new FieldDescriptor(name, type, typeId);
+
+            descs.add(fieldDescriptor);
+        }
+
+        return descs;
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code ANY-ACCESS-MODIFIER Object writeReplace() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getWriteReplace(Class<? extends Serializable> clazz) {
+        try {
+            Method writeReplace = clazz.getDeclaredMethod("writeReplace");
+
+            if (!methodThrowsExceptions(writeReplace,
+                    Collections.singletonList(ObjectStreamException.class))) {
+                return null;
+            }
+
+            return writeReplace;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code ANY-ACCESS-MODIFIER Object readResolve() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getReadResolve(Class<? extends Serializable> clazz) {
+        try {
+            Method readResolve = clazz.getDeclaredMethod("readResolve");
+
+            if (!methodThrowsExceptions(readResolve,
+                    Collections.singletonList(ObjectStreamException.class))) {
+                return null;
+            }
+
+            return readResolve;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code private void writeObject(java.io.ObjectOutputStream out) throws IOException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getWriteObject(Class<? extends Serializable> clazz) {
+        try {
+            Method writeObject = clazz.getDeclaredMethod("writeObject", ObjectOutputStream.class);
+
+            if (!Modifier.isPrivate(writeObject.getModifiers())) {
+                return null;
+            }
+
+            if (!methodThrowsExceptions(writeObject,
+                    Collections.singletonList(IOException.class))) {
+                return null;
+            }
+
+            return writeObject;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code private void readObject(java.io.ObjectInputStream in) throws IOException,
+     * ClassNotFoundException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getReadObject(Class<? extends Serializable> clazz) {
+        try {
+            Method writeObject = clazz.getDeclaredMethod("readObject", ObjectInputStream.class);
+
+            if (!Modifier.isPrivate(writeObject.getModifiers())) {
+                return null;
+            }
+
+            if (!methodThrowsExceptions(writeObject,
+                    List.of(IOException.class, ClassNotFoundException.class))) {
+                return null;
+            }
+
+            return writeObject;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code private void readObjectNoData() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getReadObjectNoData(Class<? extends Serializable> clazz) {
+        try {
+            Method writeObject = clazz.getDeclaredMethod("readObjectNoData");
+
+            if (!Modifier.isPrivate(writeObject.getModifiers())) {
+                return null;
+            }
+
+            if (!methodThrowsExceptions(writeObject,
+                    Collections.singletonList(ObjectStreamException.class))) {
+                return null;
+            }
+
+            return writeObject;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Returns {@code true} if the method throws specified exceptions.
+     *
+     * @param method Method.
+     * @param exceptions List of exceptions.
+     * @return If the method throws exceptions.
+     */
+    @SuppressWarnings("BooleanMethodIsAlwaysInverted")
+    private boolean methodThrowsExceptions(Method method,

Review comment:
       Would `methodDeclaresExactlyExceptions` be a more accurate name?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);
+        Method readResolve = getReadResolve(clazz);
+
+        SerializationType type;
+
+        if (overrideSerialization) {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        } else {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        }
+
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            type,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the arbitrary class (not serializable or externalizable) definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Arbitrary class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor arbitrary(int descriptorId, Class<?> clazz) {
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            SerializationType.ARBITRARY,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Gets field descriptors of the class. If a field's type doesn't have an id yet, generates it.
+     *
+     * @param clazz Class.
+     * @return List of field descriptor.
+     */
+    private List<FieldDescriptor> fields(Class<?> clazz) {
+        if (clazz.getSuperclass() != Object.class) {
+            // TODO: IGNITE-15945 add support for the inheritance
+            throw new UnsupportedOperationException("IGNITE-15945");
+        }
+
+        Field[] fields = clazz.getDeclaredFields();
+
+        List<FieldDescriptor> descs = new ArrayList<>(fields.length);
+
+        for (Field field : fields) {
+            Class<?> type = field.getType();
+
+            int modifiers = field.getModifiers();
+
+            // Ignore static and transient field.
+            if (Modifier.isStatic(modifiers) || Modifier.isTransient(modifiers)) {
+                continue;
+            }
+
+            String name = field.getName();
+
+            // Get or create type id.
+            int typeId = context.getId(type);
+
+            FieldDescriptor fieldDescriptor = new FieldDescriptor(name, type, typeId);
+
+            descs.add(fieldDescriptor);
+        }
+
+        return descs;
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code ANY-ACCESS-MODIFIER Object writeReplace() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getWriteReplace(Class<? extends Serializable> clazz) {
+        try {
+            Method writeReplace = clazz.getDeclaredMethod("writeReplace");
+
+            if (!methodThrowsExceptions(writeReplace,
+                    Collections.singletonList(ObjectStreamException.class))) {
+                return null;
+            }
+
+            return writeReplace;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code ANY-ACCESS-MODIFIER Object readResolve() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getReadResolve(Class<? extends Serializable> clazz) {
+        try {
+            Method readResolve = clazz.getDeclaredMethod("readResolve");
+
+            if (!methodThrowsExceptions(readResolve,
+                    Collections.singletonList(ObjectStreamException.class))) {
+                return null;
+            }
+
+            return readResolve;
+        } catch (NoSuchMethodException e) {
+            return null;
+        }
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code private void writeObject(java.io.ObjectOutputStream out) throws IOException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getWriteObject(Class<? extends Serializable> clazz) {
+        try {
+            Method writeObject = clazz.getDeclaredMethod("writeObject", ObjectOutputStream.class);
+
+            if (!Modifier.isPrivate(writeObject.getModifiers())) {
+                return null;
+            }
+
+            if (!methodThrowsExceptions(writeObject,
+                    Collections.singletonList(IOException.class))) {

Review comment:
       What if someone does not declare `IOexception` because they catch it themselves, for example? Will we miss such a method?

##########
File path: modules/network/src/test/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParserTest.java
##########
@@ -0,0 +1,366 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNotNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInput;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutput;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.util.List;
+import java.util.Map;
+import java.util.function.Function;
+import java.util.stream.Collectors;
+import org.apache.ignite.internal.network.serialization.DefaultClassDescriptors.DefaultType;
+import org.apache.ignite.lang.IgniteException;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+/**
+ * Set of tests for the class descriptor parser.
+ */
+public class ClassDescriptorParserTest {
+    /**
+     * Descriptor parser context.
+     */
+    private ClassDescriptorParserContext context;
+
+    /**
+     * Descriptor parser.
+     */
+    private ClassDescriptorParser parser;
+
+    /**
+     * Set up method.
+     */
+    @BeforeEach
+    void setUp() {
+        context = new ClassDescriptorParserContext();

Review comment:
       These can be just initialized on fields:
   
   ```
   private final ClassDescriptorParserContext context = new ClassDescriptorParserContext();
   ...
   ```

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);
+        Method readResolve = getReadResolve(clazz);
+
+        SerializationType type;
+
+        if (overrideSerialization) {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        } else {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        }
+
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            type,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the arbitrary class (not serializable or externalizable) definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Arbitrary class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor arbitrary(int descriptorId, Class<?> clazz) {
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            SerializationType.ARBITRARY,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Gets field descriptors of the class. If a field's type doesn't have an id yet, generates it.
+     *
+     * @param clazz Class.
+     * @return List of field descriptor.
+     */
+    private List<FieldDescriptor> fields(Class<?> clazz) {
+        if (clazz.getSuperclass() != Object.class) {
+            // TODO: IGNITE-15945 add support for the inheritance
+            throw new UnsupportedOperationException("IGNITE-15945");
+        }
+
+        Field[] fields = clazz.getDeclaredFields();
+
+        List<FieldDescriptor> descs = new ArrayList<>(fields.length);
+
+        for (Field field : fields) {
+            Class<?> type = field.getType();
+
+            int modifiers = field.getModifiers();
+
+            // Ignore static and transient field.

Review comment:
       How about synthetic fields?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :

Review comment:
       BTW, it seems possible to return a known built-in descriptor for a primitive type instead of exploding. Is this handled on an upper level?

##########
File path: modules/network/src/main/java/org/apache/ignite/internal/network/serialization/ClassDescriptorParser.java
##########
@@ -0,0 +1,432 @@
+/*
+ * 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.ignite.internal.network.serialization;
+
+import java.io.Externalizable;
+import java.io.IOException;
+import java.io.ObjectInputStream;
+import java.io.ObjectOutputStream;
+import java.io.ObjectStreamException;
+import java.io.Serializable;
+import java.lang.reflect.Constructor;
+import java.lang.reflect.Field;
+import java.lang.reflect.Method;
+import java.lang.reflect.Modifier;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Queue;
+import org.apache.ignite.lang.IgniteException;
+
+/**
+ * Class parser for the user object serialization.
+ */
+public class ClassDescriptorParser {
+
+    /**
+     * Parser context.
+     */
+    private final ClassDescriptorParserContext context;
+
+    /**
+     * Constructor.
+     *
+     * @param ctx Context.
+     */
+    public ClassDescriptorParser(ClassDescriptorParserContext ctx) {
+        this.context = ctx;
+    }
+
+    /**
+     * Parses the class definition and definitions of classes fields if they're not already parsed.
+     *
+     * @param clazz Class definition.
+     * @return Class descriptor.
+     */
+    public ClassDescriptor parse(Class<?> clazz) {
+        ClassDescriptor classDesc = parse0(clazz);
+
+        Map<Integer, ClassDescriptor> localDescs = new HashMap<>();
+        localDescs.put(classDesc.descriptorId(), classDesc);
+
+        Queue<FieldDescriptor> fieldDescriptors = new LinkedList<>(classDesc.fields());
+
+        while (!fieldDescriptors.isEmpty()) {
+            FieldDescriptor fieldDescriptor = fieldDescriptors.remove();
+
+            int typeDescriptorId = fieldDescriptor.typeDescriptorId();
+
+            if (context.hasDescriptor(typeDescriptorId) || localDescs.containsKey(
+                    typeDescriptorId)) {
+                continue;
+            }
+
+            Class<?> fieldClass = fieldDescriptor.clazz();
+
+            ClassDescriptor fieldClassDesc = parse0(fieldClass);
+
+            localDescs.put(typeDescriptorId, fieldClassDesc);
+
+            fieldDescriptors.addAll(fieldClassDesc.fields());
+        }
+
+        context.addDescriptors(localDescs.values());
+
+        return classDesc;
+    }
+
+    /**
+     * Parses the class definition.
+     *
+     * @param clazz Class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor parse0(Class<?> clazz) {
+        assert !clazz.isPrimitive() :
+            clazz + " is a primitive, there should be a default descriptor";
+
+        int descriptorId = context.getId(clazz);
+
+        if (Externalizable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return externalizable(descriptorId, (Class<? extends Externalizable>) clazz);
+        } else if (Serializable.class.isAssignableFrom(clazz)) {
+            //noinspection unchecked
+            return serializable(descriptorId, (Class<? extends Serializable>) clazz);
+        }
+
+        return arbitrary(descriptorId, clazz);
+    }
+
+    /**
+     * Parses the externalizable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz        Externalizable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor externalizable(int descriptorId,
+            Class<? extends Externalizable> clazz) {
+        String className = className(clazz);
+
+        boolean hasPublicNoArgConstructor = true;
+
+        try {
+            Constructor<? extends Externalizable> ctor = clazz.getConstructor();
+
+            if (!Modifier.isPublic(ctor.getModifiers())) {
+                hasPublicNoArgConstructor = false;
+            }
+        } catch (NoSuchMethodException e) {
+            hasPublicNoArgConstructor = false;
+        }
+
+        if (!hasPublicNoArgConstructor) {
+            throw new IgniteException(
+                "Externalizable class " + className + " has no public no-arg constructor");
+        }
+
+        return new ClassDescriptor(
+            className,
+            descriptorId,
+            Collections.emptyList(),
+            SerializationType.EXTERNALIZABLE,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the serializable class definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Serializable class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor serializable(int descriptorId, Class<? extends Serializable> clazz) {
+        Method writeObject = getWriteObject(clazz);
+        Method readObject = getReadObject(clazz);
+        Method readObjectNoData = getReadObjectNoData(clazz);
+
+        boolean overrideSerialization = false;
+
+        if (writeObject != null || readObject != null || readObjectNoData != null) {
+            if (writeObject == null || readObject == null || readObjectNoData == null) {
+                throw new IgniteException("!");
+            }
+
+            overrideSerialization = true;
+        }
+
+        Method writeReplace = getWriteReplace(clazz);
+        Method readResolve = getReadResolve(clazz);
+
+        SerializationType type;
+
+        if (overrideSerialization) {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_OVERRIDE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        } else {
+            if (writeReplace == null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE;
+            } else if (writeReplace != null && readResolve == null) {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE;
+            } else if (writeReplace == null) {
+                type = SerializationType.SERIALIZABLE_READ_RESOLVE;
+            } else {
+                type = SerializationType.SERIALIZABLE_WRITE_REPLACE_READ_RESOLVE;
+            }
+        }
+
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            type,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Parses the arbitrary class (not serializable or externalizable) definition.
+     *
+     * @param descriptorId Descriptor id of the class.
+     * @param clazz Arbitrary class.
+     * @return Class descriptor.
+     */
+    private ClassDescriptor arbitrary(int descriptorId, Class<?> clazz) {
+        return new ClassDescriptor(
+            className(clazz),
+            descriptorId,
+            fields(clazz),
+            SerializationType.ARBITRARY,
+            isFinal(clazz)
+        );
+    }
+
+    /**
+     * Gets field descriptors of the class. If a field's type doesn't have an id yet, generates it.
+     *
+     * @param clazz Class.
+     * @return List of field descriptor.
+     */
+    private List<FieldDescriptor> fields(Class<?> clazz) {
+        if (clazz.getSuperclass() != Object.class) {
+            // TODO: IGNITE-15945 add support for the inheritance
+            throw new UnsupportedOperationException("IGNITE-15945");
+        }
+
+        Field[] fields = clazz.getDeclaredFields();
+
+        List<FieldDescriptor> descs = new ArrayList<>(fields.length);
+
+        for (Field field : fields) {
+            Class<?> type = field.getType();
+
+            int modifiers = field.getModifiers();
+
+            // Ignore static and transient field.
+            if (Modifier.isStatic(modifiers) || Modifier.isTransient(modifiers)) {
+                continue;
+            }
+
+            String name = field.getName();
+
+            // Get or create type id.
+            int typeId = context.getId(type);
+
+            FieldDescriptor fieldDescriptor = new FieldDescriptor(name, type, typeId);
+
+            descs.add(fieldDescriptor);
+        }
+
+        return descs;
+    }
+
+    /**
+     * Gets a method with the signature
+     * {@code ANY-ACCESS-MODIFIER Object writeReplace() throws ObjectStreamException}.
+     *
+     * @param clazz Class.
+     * @return Method.
+     */
+    private Method getWriteReplace(Class<? extends Serializable> clazz) {
+        try {
+            Method writeReplace = clazz.getDeclaredMethod("writeReplace");
+
+            if (!methodThrowsExceptions(writeReplace,
+                    Collections.singletonList(ObjectStreamException.class))) {

Review comment:
       Why do we need this specific exception?




-- 
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: notifications-unsubscribe@ignite.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org