You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by sb...@apache.org on 2015/08/25 09:05:50 UTC

[09/14] ignite git commit: ignite-1258: portable objects API support in Ignite

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableUtils.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableUtils.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableUtils.java
new file mode 100644
index 0000000..72e80ec
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableUtils.java
@@ -0,0 +1,419 @@
+/*
+ * 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.portable;
+
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.portable.*;
+
+import org.jetbrains.annotations.*;
+import org.jsr166.*;
+
+import java.math.*;
+import java.sql.*;
+import java.util.*;
+import java.util.Date;
+import java.util.concurrent.*;
+
+import static org.apache.ignite.internal.portable.GridPortableMarshaller.*;
+
+/**
+ *
+ */
+public class PortableUtils {
+    /** */
+    public static final Map<Class<?>, Byte> PLAIN_CLASS_TO_FLAG = new HashMap<>();
+
+    /** */
+    public static final Map<Byte, Class<?>> FLAG_TO_CLASS = new HashMap<>();
+
+    /** {@code true} if serialized value of this type cannot contain references to objects. */
+    private static final boolean[] PLAIN_TYPE_FLAG = new boolean[102];
+
+    /** Portable classes. */
+    private static final Collection<Class<?>> PORTABLE_CLS = new HashSet<>();
+
+    /**
+     *
+     */
+    static {
+        PORTABLE_CLS.add(Byte.class);
+        PORTABLE_CLS.add(Short.class);
+        PORTABLE_CLS.add(Integer.class);
+        PORTABLE_CLS.add(Long.class);
+        PORTABLE_CLS.add(Float.class);
+        PORTABLE_CLS.add(Double.class);
+        PORTABLE_CLS.add(Character.class);
+        PORTABLE_CLS.add(Boolean.class);
+        PORTABLE_CLS.add(String.class);
+        PORTABLE_CLS.add(UUID.class);
+        PORTABLE_CLS.add(Date.class);
+        PORTABLE_CLS.add(Timestamp.class);
+        PORTABLE_CLS.add(BigDecimal.class);
+        PORTABLE_CLS.add(byte[].class);
+        PORTABLE_CLS.add(short[].class);
+        PORTABLE_CLS.add(int[].class);
+        PORTABLE_CLS.add(long[].class);
+        PORTABLE_CLS.add(float[].class);
+        PORTABLE_CLS.add(double[].class);
+        PORTABLE_CLS.add(char[].class);
+        PORTABLE_CLS.add(boolean[].class);
+        PORTABLE_CLS.add(String[].class);
+        PORTABLE_CLS.add(UUID[].class);
+        PORTABLE_CLS.add(Date[].class);
+        PORTABLE_CLS.add(Timestamp[].class);
+        PORTABLE_CLS.add(BigDecimal[].class);
+    }
+
+    /**
+     *
+     */
+    static {
+        PLAIN_CLASS_TO_FLAG.put(Byte.class, GridPortableMarshaller.BYTE);
+        PLAIN_CLASS_TO_FLAG.put(Short.class, GridPortableMarshaller.SHORT);
+        PLAIN_CLASS_TO_FLAG.put(Integer.class, GridPortableMarshaller.INT);
+        PLAIN_CLASS_TO_FLAG.put(Long.class, GridPortableMarshaller.LONG);
+        PLAIN_CLASS_TO_FLAG.put(Float.class, GridPortableMarshaller.FLOAT);
+        PLAIN_CLASS_TO_FLAG.put(Double.class, GridPortableMarshaller.DOUBLE);
+        PLAIN_CLASS_TO_FLAG.put(Character.class, GridPortableMarshaller.CHAR);
+        PLAIN_CLASS_TO_FLAG.put(Boolean.class, GridPortableMarshaller.BOOLEAN);
+        PLAIN_CLASS_TO_FLAG.put(BigDecimal.class, GridPortableMarshaller.DECIMAL);
+        PLAIN_CLASS_TO_FLAG.put(String.class, GridPortableMarshaller.STRING);
+        PLAIN_CLASS_TO_FLAG.put(UUID.class, GridPortableMarshaller.UUID);
+        PLAIN_CLASS_TO_FLAG.put(Date.class, GridPortableMarshaller.DATE);
+
+        PLAIN_CLASS_TO_FLAG.put(byte[].class, GridPortableMarshaller.BYTE_ARR);
+        PLAIN_CLASS_TO_FLAG.put(short[].class, GridPortableMarshaller.SHORT_ARR);
+        PLAIN_CLASS_TO_FLAG.put(int[].class, GridPortableMarshaller.INT_ARR);
+        PLAIN_CLASS_TO_FLAG.put(long[].class, GridPortableMarshaller.LONG_ARR);
+        PLAIN_CLASS_TO_FLAG.put(float[].class, GridPortableMarshaller.FLOAT_ARR);
+        PLAIN_CLASS_TO_FLAG.put(double[].class, GridPortableMarshaller.DOUBLE_ARR);
+        PLAIN_CLASS_TO_FLAG.put(char[].class, GridPortableMarshaller.CHAR_ARR);
+        PLAIN_CLASS_TO_FLAG.put(boolean[].class, GridPortableMarshaller.BOOLEAN_ARR);
+        PLAIN_CLASS_TO_FLAG.put(BigDecimal[].class, GridPortableMarshaller.DECIMAL_ARR);
+        PLAIN_CLASS_TO_FLAG.put(String[].class, GridPortableMarshaller.STRING_ARR);
+        PLAIN_CLASS_TO_FLAG.put(UUID[].class, GridPortableMarshaller.UUID_ARR);
+        PLAIN_CLASS_TO_FLAG.put(Date[].class, GridPortableMarshaller.DATE_ARR);
+
+        for (Map.Entry<Class<?>, Byte> entry : PLAIN_CLASS_TO_FLAG.entrySet())
+            FLAG_TO_CLASS.put(entry.getValue(), entry.getKey());
+
+        PLAIN_CLASS_TO_FLAG.put(byte.class, GridPortableMarshaller.BYTE);
+        PLAIN_CLASS_TO_FLAG.put(short.class, GridPortableMarshaller.SHORT);
+        PLAIN_CLASS_TO_FLAG.put(int.class, GridPortableMarshaller.INT);
+        PLAIN_CLASS_TO_FLAG.put(long.class, GridPortableMarshaller.LONG);
+        PLAIN_CLASS_TO_FLAG.put(float.class, GridPortableMarshaller.FLOAT);
+        PLAIN_CLASS_TO_FLAG.put(double.class, GridPortableMarshaller.DOUBLE);
+        PLAIN_CLASS_TO_FLAG.put(char.class, GridPortableMarshaller.CHAR);
+        PLAIN_CLASS_TO_FLAG.put(boolean.class, GridPortableMarshaller.BOOLEAN);
+
+        for (byte b : new byte[] {
+            BYTE, SHORT, INT, LONG, FLOAT, DOUBLE,
+            CHAR, BOOLEAN, DECIMAL, STRING, UUID, DATE,
+            BYTE_ARR, SHORT_ARR, INT_ARR, LONG_ARR, FLOAT_ARR, DOUBLE_ARR,
+            CHAR_ARR, BOOLEAN_ARR, DECIMAL_ARR, STRING_ARR, UUID_ARR, DATE_ARR,
+            ENUM, ENUM_ARR, NULL}) {
+
+            PLAIN_TYPE_FLAG[b] = true;
+        }
+    }
+
+    /**
+     * Write value with flag. e.g. writePlainObject(writer, (byte)77) will write two byte: {BYTE, 77}.
+     *
+     * @param writer W
+     * @param val Value.
+     */
+    public static void writePlainObject(PortableWriterExImpl writer, Object val) {
+        Byte flag = PLAIN_CLASS_TO_FLAG.get(val.getClass());
+
+        if (flag == null)
+            throw new IllegalArgumentException("Can't write object with type: " + val.getClass());
+
+        switch (flag) {
+            case BYTE:
+                writer.writeByte(flag);
+                writer.writeByte((Byte)val);
+
+                break;
+
+            case SHORT:
+                writer.writeByte(flag);
+                writer.writeShort((Short)val);
+
+                break;
+
+            case INT:
+                writer.writeByte(flag);
+                writer.writeInt((Integer)val);
+
+                break;
+
+            case LONG:
+                writer.writeByte(flag);
+                writer.writeLong((Long)val);
+
+                break;
+
+            case FLOAT:
+                writer.writeByte(flag);
+                writer.writeFloat((Float)val);
+
+                break;
+
+            case DOUBLE:
+                writer.writeByte(flag);
+                writer.writeDouble((Double)val);
+
+                break;
+
+            case CHAR:
+                writer.writeByte(flag);
+                writer.writeChar((Character)val);
+
+                break;
+
+            case BOOLEAN:
+                writer.writeByte(flag);
+                writer.writeBoolean((Boolean)val);
+
+                break;
+
+            case DECIMAL:
+                writer.doWriteDecimal((BigDecimal)val);
+
+                break;
+
+            case STRING:
+                writer.doWriteString((String)val);
+
+                break;
+
+            case UUID:
+                writer.doWriteUuid((UUID)val);
+
+                break;
+
+            case DATE:
+                if (val instanceof Timestamp)
+                    writer.doWriteTimestamp((Timestamp)val);
+                else
+                    writer.doWriteDate((Date)val);
+
+                break;
+
+            case BYTE_ARR:
+                writer.doWriteByteArray((byte[])val);
+
+                break;
+
+            case SHORT_ARR:
+                writer.doWriteShortArray((short[])val);
+
+                break;
+
+            case INT_ARR:
+                writer.doWriteIntArray((int[])val);
+
+                break;
+
+            case LONG_ARR:
+                writer.doWriteLongArray((long[])val);
+
+                break;
+
+            case FLOAT_ARR:
+                writer.doWriteFloatArray((float[])val);
+
+                break;
+
+            case DOUBLE_ARR:
+                writer.doWriteDoubleArray((double[])val);
+
+                break;
+
+            case CHAR_ARR:
+                writer.doWriteCharArray((char[])val);
+
+                break;
+
+            case BOOLEAN_ARR:
+                writer.doWriteBooleanArray((boolean[])val);
+
+                break;
+
+            case DECIMAL_ARR:
+                writer.doWriteDecimalArray((BigDecimal[])val);
+
+                break;
+
+            case STRING_ARR:
+                writer.doWriteStringArray((String[])val);
+
+                break;
+
+            case UUID_ARR:
+                writer.doWriteUuidArray((UUID[])val);
+
+                break;
+
+            case DATE_ARR:
+                writer.doWriteDateArray((Date[])val);
+
+                break;
+
+            default:
+                throw new IllegalArgumentException("Can't write object with type: " + val.getClass());
+        }
+    }
+
+    /**
+     * @param obj Value to unwrap.
+     * @return Unwrapped value.
+     */
+    public static Object unwrapLazy(@Nullable Object obj) {
+        if (obj instanceof PortableLazyValue)
+            return ((PortableLazyValue)obj).value();
+
+        return obj;
+    }
+
+    /**
+     * @param delegate Iterator to delegate.
+     * @return New iterator.
+     */
+    public static Iterator<Object> unwrapLazyIterator(final Iterator<Object> delegate) {
+        return new Iterator<Object>() {
+            @Override public boolean hasNext() {
+                return delegate.hasNext();
+            }
+
+            @Override public Object next() {
+                return unwrapLazy(delegate.next());
+            }
+
+            @Override public void remove() {
+                delegate.remove();
+            }
+        };
+    }
+
+    /**
+     * @return {@code true} if content of serialized value cannot contain references to other object.
+     */
+    public static boolean isPlainType(int type) {
+        return type > 0 && type < PLAIN_TYPE_FLAG.length && PLAIN_TYPE_FLAG[type];
+    }
+
+    /**
+     * @param cls Class.
+     * @return Portable field type.
+     */
+    public static byte typeByClass(Class<?> cls) {
+        if (Date.class.isAssignableFrom(cls))
+            return DATE;
+
+        Byte type = PLAIN_CLASS_TO_FLAG.get(cls);
+
+        if (type != null)
+            return type;
+
+        if (cls.isEnum())
+            return ENUM;
+
+        if (cls.isArray())
+            return cls.getComponentType().isEnum() || cls.getComponentType() == Enum.class ? ENUM_ARR : OBJ_ARR;
+
+        if (Collection.class.isAssignableFrom(cls))
+            return COL;
+
+        if (Map.class.isAssignableFrom(cls))
+            return MAP;
+
+        if (Map.Entry.class.isAssignableFrom(cls))
+            return MAP;
+
+        return OBJ;
+    }
+
+    /**
+     * Tells whether provided type is portable or a collection.
+     *
+     * @param cls Class to check.
+     * @return Whether type is portable or a collection.
+     */
+    public static boolean isPortableOrCollectionType(Class<?> cls) {
+        assert cls != null;
+
+        return isPortableType(cls) ||
+            cls == Object[].class ||
+            Collection.class.isAssignableFrom(cls) ||
+            Map.class.isAssignableFrom(cls) ||
+            Map.Entry.class.isAssignableFrom(cls);
+    }
+
+    /**
+     * Tells whether provided type is portable.
+     *
+     * @param cls Class to check.
+     * @return Whether type is portable.
+     */
+    public static boolean isPortableType(Class<?> cls) {
+        assert cls != null;
+
+        return PortableObject.class.isAssignableFrom(cls) ||
+            PORTABLE_CLS.contains(cls) ||
+            cls.isEnum() ||
+            (cls.isArray() && cls.getComponentType().isEnum());
+    }
+
+    /**
+     * Attempts to create a new map of the same type as {@code map} has. Otherwise returns new {@code HashMap} instance.
+     *
+     * @param map Original map.
+     * @return New map.
+     */
+    public static <K, V> Map<K, V> newMap(Map<K, V> map) {
+        if (map instanceof LinkedHashMap)
+            return U.newLinkedHashMap(map.size());
+        else if (map instanceof TreeMap)
+            return new TreeMap<>(((TreeMap<Object, Object>)map).comparator());
+        else if (map instanceof ConcurrentHashMap8)
+            return new ConcurrentHashMap8<>(U.capacity(map.size()));
+        else if (map instanceof ConcurrentHashMap)
+            return new ConcurrentHashMap<>(U.capacity(map.size()));
+
+        return U.newHashMap(map.size());
+    }
+
+    /**
+     * Attempts to create a new set of the same type as {@code set} has. Otherwise returns new {@code HashSet} instance.
+     *
+     * @param set Original set.
+     * @return New set.
+     */
+    public static <V> Set<V> newSet(Set<V> set) {
+        if (set instanceof LinkedHashSet)
+            return U.newLinkedHashSet(set.size());
+        else if (set instanceof TreeSet)
+            return new TreeSet<>(((TreeSet<Object>)set).comparator());
+        else if (set instanceof ConcurrentSkipListSet)
+            return new ConcurrentSkipListSet<>(((ConcurrentSkipListSet<Object>)set).comparator());
+
+        return U.newHashSet(set.size());
+    }
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableValueWithType.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableValueWithType.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableValueWithType.java
new file mode 100644
index 0000000..52f1dcc
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableValueWithType.java
@@ -0,0 +1,74 @@
+/*
+ * 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.portable;
+
+import org.apache.ignite.internal.processors.cache.portable.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+
+/**
+ *
+ */
+class PortableValueWithType implements PortableLazyValue {
+    /** */
+    private byte type;
+
+    /** */
+    private Object val;
+
+    /**
+     * @param type Type
+     * @param val Value.
+     */
+    PortableValueWithType(byte type, Object val) {
+        this.type = type;
+        this.val = val;
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeTo(PortableWriterExImpl writer, PortableBuilderSerializer ctx) {
+        if (val instanceof PortableBuilderSerializationAware)
+            ((PortableBuilderSerializationAware)val).writeTo(writer, ctx);
+        else
+            ctx.writeValue(writer, val);
+    }
+
+    /** {@inheritDoc} */
+    public String typeName() {
+        return CacheObjectPortableProcessorImpl.fieldTypeName(type);
+    }
+
+    /** {@inheritDoc} */
+    @Override public Object value() {
+        if (val instanceof PortableLazyValue)
+            return ((PortableLazyValue)val).value();
+
+        return val;
+    }
+
+    /**
+     * @param val New value.
+     */
+    public void value(Object val) {
+        this.val = val;
+    }
+
+    /** {@inheritDoc} */
+    @Override public String toString() {
+        return S.toString(PortableValueWithType.class, this);
+    }
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableWriterExImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableWriterExImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableWriterExImpl.java
new file mode 100644
index 0000000..f801632
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/PortableWriterExImpl.java
@@ -0,0 +1,1769 @@
+/*
+ * 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.portable;
+
+import org.apache.ignite.*;
+import org.apache.ignite.internal.portable.streams.*;
+import org.apache.ignite.internal.util.typedef.internal.*;
+import org.apache.ignite.portable.*;
+
+import org.jetbrains.annotations.*;
+
+import java.io.*;
+import java.lang.reflect.*;
+import java.math.*;
+import java.sql.*;
+import java.util.*;
+import java.util.Date;
+import java.util.concurrent.*;
+
+import static java.nio.charset.StandardCharsets.*;
+import static org.apache.ignite.internal.portable.GridPortableMarshaller.*;
+
+ /**
+ * Portable writer implementation.
+ */
+public class PortableWriterExImpl implements PortableWriter, PortableRawWriterEx, ObjectOutput {
+    /** Length: integer. */
+    private static final int LEN_INT = 4;
+
+    /** */
+    private static final int INIT_CAP = 1024;
+
+    /** */
+    private static final ConcurrentHashMap<Class<?>, Boolean> useOptMarshCache = new ConcurrentHashMap<>();
+
+    /** */
+    private final PortableContext ctx;
+
+    /** */
+    private final WriterContext wCtx;
+
+    /** */
+    private final int start;
+
+    /** */
+    private int mark;
+
+    /** */
+    private Class<?> cls;
+
+    /** */
+    private int typeId;
+
+    /** */
+    private boolean allowFields = true;
+
+    /** */
+    private boolean metaEnabled;
+
+    /** */
+    private int metaHashSum;
+
+    /**
+     * @param ctx Context.
+     * @param off Start offset.
+     */
+    PortableWriterExImpl(PortableContext ctx, int off) {
+        this.ctx = ctx;
+
+        PortableOutputStream out = new PortableHeapOutputStream(off + INIT_CAP);
+
+        out.position(off);
+
+        wCtx = new WriterContext(out, null);
+
+        start = off;
+    }
+
+    /**
+     * @param ctx Context.
+     * @param out Output stream.
+     * @param off Start offset.
+     */
+    PortableWriterExImpl(PortableContext ctx, PortableOutputStream out, int off) {
+        this.ctx = ctx;
+
+        wCtx = new WriterContext(out, null);
+
+        start = off;
+    }
+
+    /**
+     * @param ctx Context.
+     * @param off Start offset.
+     * @param typeId Type ID.
+     */
+    PortableWriterExImpl(PortableContext ctx, int off, int typeId, boolean metaEnabled) {
+        this(ctx, off);
+
+        this.typeId = typeId;
+
+        this.metaEnabled = metaEnabled;
+    }
+
+    /**
+     * @param ctx Context.
+     * @param wCtx Writer context.
+     */
+    private PortableWriterExImpl(PortableContext ctx, WriterContext wCtx) {
+        this.ctx = ctx;
+        this.wCtx = wCtx;
+
+        start = wCtx.out.position();
+    }
+
+    /**
+     * Close the writer releasing resources if necessary.
+     */
+    @Override public void close() {
+        wCtx.out.close();
+    }
+
+    /**
+     * @return Meta data hash sum or {@code null} if meta data is disabled.
+     */
+    @Nullable Integer metaDataHashSum() {
+        return metaEnabled ? metaHashSum : null;
+    }
+
+    /**
+     * @param obj Object.
+     * @param detached Detached or not.
+     * @throws PortableException In case of error.
+     */
+    void marshal(Object obj, boolean detached) throws PortableException {
+        assert obj != null;
+
+        if (useOptimizedMarshaller(obj)) {
+            writeByte(OPTM_MARSH);
+
+            try {
+                byte[] arr = ctx.optimizedMarsh().marshal(obj);
+
+                writeInt(arr.length);
+
+                write(arr);
+            }
+            catch (IgniteCheckedException e) {
+                throw new PortableException("Failed to marshal object with optimized marshaller: " + obj, e);
+            }
+
+            return;
+        }
+
+        cls = obj.getClass();
+
+        PortableClassDescriptor desc = ctx.descriptorForClass(cls);
+
+        if (desc == null)
+            throw new PortableException("Object is not portable: [class=" + cls + ']');
+
+        if (desc.excluded()) {
+            doWriteByte(NULL);
+            return;
+        }
+
+        if (desc.getWriteReplaceMethod() != null) {
+            Object replace;
+
+            try {
+                replace = desc.getWriteReplaceMethod().invoke(obj);
+            }
+            catch (IllegalAccessException e) {
+                throw new RuntimeException(e);
+            }
+            catch (InvocationTargetException e) {
+                if (e.getTargetException() instanceof PortableException)
+                    throw (PortableException)e.getTargetException();
+
+                throw new PortableException("Failed to execute writeReplace() method on " + obj, e);
+            }
+
+            if (replace == null) {
+                doWriteByte(NULL);
+                return;
+            }
+
+            if (cls != replace.getClass()) {
+                cls = replace.getClass();
+
+                desc = ctx.descriptorForClass(cls);
+
+                if (desc == null)
+                    throw new PortableException("Object is not portable: [class=" + cls + ']');
+            }
+
+            obj = replace;
+        }
+
+        typeId = desc.typeId();
+
+        metaEnabled = ctx.isMetaDataEnabled(typeId);
+
+        if (detached)
+            wCtx.resetHandles();
+
+        desc.write(obj, this);
+    }
+
+     /**
+      * Determines whether to use {@link org.apache.ignite.marshaller.optimized.OptimizedMarshaller} for serialization
+      * or not.
+      *
+      * @param obj Object to serialize.
+      * @return {@code true} if to use, {@code false} otherwise.
+      */
+     private boolean useOptimizedMarshaller(Object obj) {
+         Class<?> cls = obj.getClass();
+
+         Boolean use = useOptMarshCache.get(cls);
+
+         if (use != null)
+             return use;
+
+         if (ctx.isPredefinedClass(cls))
+             use = false;
+         else {
+             try {
+                 Method writeObj = cls.getDeclaredMethod("writeObject", ObjectOutputStream.class);
+                 Method readObj = cls.getDeclaredMethod("readObject", ObjectInputStream.class);
+
+                 if (!Modifier.isStatic(writeObj.getModifiers()) && !Modifier.isStatic(readObj.getModifiers()) &&
+                     writeObj.getReturnType() == void.class && readObj.getReturnType() == void.class)
+                     use = true;
+                 else
+                     use = false;
+
+             } catch (NoSuchMethodException e) {
+                 use = false;
+             }
+         }
+
+         useOptMarshCache.putIfAbsent(cls, use);
+
+         return use;
+     }
+
+    /**
+     * @param obj Object.
+     * @return Handle.
+     */
+    int handle(Object obj) {
+        assert obj != null;
+
+        return wCtx.handle(obj);
+    }
+
+    /**
+     * @return Array.
+     */
+    byte[] array() {
+        return wCtx.out.arrayCopy();
+    }
+
+    /**
+     * @return Output stream.
+     */
+    PortableOutputStream outputStream() {
+        return wCtx.out;
+    }
+
+    /**
+     * @return Stream current position.
+     */
+    int position() {
+        return wCtx.out.position();
+    }
+
+    /**
+     * Sets new position.
+     *
+     * @param pos Position.
+     */
+    void position(int pos) {
+        wCtx.out.position(pos);
+    }
+
+     /**
+     * @param bytes Number of bytes to reserve.
+     * @return Offset.
+     */
+    int reserve(int bytes) {
+        int pos = wCtx.out.position();
+
+        wCtx.out.position(pos + bytes);
+
+        return pos;
+    }
+
+    /**
+     * @param bytes Number of bytes to reserve.
+     * @return Offset.
+     */
+    int reserveAndMark(int bytes) {
+        int off0 = reserve(bytes);
+
+        mark = wCtx.out.position();
+
+        return off0;
+    }
+
+    /**
+     * @param off Offset.
+     */
+    void writeDelta(int off) {
+        wCtx.out.writeInt(off, wCtx.out.position() - mark);
+    }
+
+    /**
+     *
+     */
+    void writeLength() {
+        wCtx.out.writeInt(start + TOTAL_LEN_POS, wCtx.out.position() - start);
+    }
+
+    /**
+     *
+     */
+    void writeRawOffsetIfNeeded() {
+        if (allowFields)
+            wCtx.out.writeInt(start + RAW_DATA_OFF_POS, wCtx.out.position() - start);
+    }
+
+    /**
+     * @param val Byte array.
+     */
+    public void write(byte[] val) {
+        assert val != null;
+
+        wCtx.out.writeByteArray(val);
+    }
+
+    /**
+     * @param val Byte array.
+     * @param off Offset.
+     * @param len Length.
+     */
+    public void write(byte[] val, int off, int len) {
+        assert val != null;
+
+        wCtx.out.write(val, off, len);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteByte(byte val) {
+        wCtx.out.writeByte(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteShort(short val) {
+        wCtx.out.writeShort(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteInt(int val) {
+        wCtx.out.writeInt(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteLong(long val) {
+        wCtx.out.writeLong(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteFloat(float val) {
+        wCtx.out.writeFloat(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteDouble(double val) {
+        wCtx.out.writeDouble(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteChar(char val) {
+        wCtx.out.writeChar(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteBoolean(boolean val) {
+        wCtx.out.writeBoolean(val);
+    }
+
+    /**
+     * @param val String value.
+     */
+    void doWriteDecimal(@Nullable BigDecimal val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DECIMAL);
+
+            BigInteger intVal = val.unscaledValue();
+
+            if (intVal.signum() == -1) {
+                intVal = intVal.negate();
+
+                wCtx.out.writeInt(val.scale() | 0x80000000);
+            }
+            else
+                wCtx.out.writeInt(val.scale());
+
+            byte[] vals = intVal.toByteArray();
+
+            wCtx.out.writeInt(vals.length);
+            wCtx.out.writeByteArray(vals);
+        }
+    }
+
+    /**
+     * @param val String value.
+     */
+    void doWriteString(@Nullable String val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(STRING);
+
+            if (ctx.isConvertString()) {
+                doWriteBoolean(true);
+
+                byte[] strArr = val.getBytes(UTF_8);
+
+                doWriteInt(strArr.length);
+
+                wCtx.out.writeByteArray(strArr);
+            }
+            else {
+                doWriteBoolean(false);
+
+                char[] strArr = val.toCharArray();
+
+                doWriteInt(strArr.length);
+
+                wCtx.out.writeCharArray(strArr);
+            }
+        }
+    }
+
+    /**
+     * @param uuid UUID.
+     */
+    void doWriteUuid(@Nullable UUID uuid) {
+        if (uuid == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(UUID);
+            doWriteLong(uuid.getMostSignificantBits());
+            doWriteLong(uuid.getLeastSignificantBits());
+        }
+    }
+
+    /**
+     * @param date Date.
+     */
+    void doWriteDate(@Nullable Date date) {
+        if (date == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DATE);
+            doWriteLong(date.getTime());
+            doWriteInt(0);
+        }
+    }
+
+    /**
+     * @param ts Timestamp.
+     */
+    void doWriteTimestamp(@Nullable Timestamp ts) {
+        if (ts == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DATE);
+            doWriteLong(ts.getTime());
+            doWriteInt(ts.getNanos() % 1000000);
+        }
+    }
+
+    /**
+     * @param obj Object.
+     * @param detached Detached or not.
+     * @throws PortableException In case of error.
+     */
+    void doWriteObject(@Nullable Object obj, boolean detached) throws PortableException {
+        if (obj == null)
+            doWriteByte(NULL);
+        else {
+            WriterContext wCtx = detached ? new WriterContext(this.wCtx.out, this.wCtx.handles) : this.wCtx;
+
+            PortableWriterExImpl writer = new PortableWriterExImpl(ctx, wCtx);
+
+            writer.marshal(obj, detached);
+
+            if (detached)
+                this.wCtx.out = wCtx.out;
+        }
+    }
+
+    /**
+     * @param val Byte array.
+     */
+    void doWriteByteArray(@Nullable byte[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(BYTE_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeByteArray(val);
+        }
+    }
+
+    /**
+     * @param val Short array.
+     */
+    void doWriteShortArray(@Nullable short[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(SHORT_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeShortArray(val);
+        }
+    }
+
+    /**
+     * @param val Integer array.
+     */
+    void doWriteIntArray(@Nullable int[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(INT_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeIntArray(val);
+        }
+    }
+
+    /**
+     * @param val Long array.
+     */
+    void doWriteLongArray(@Nullable long[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(LONG_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeLongArray(val);
+        }
+    }
+
+    /**
+     * @param val Float array.
+     */
+    void doWriteFloatArray(@Nullable float[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(FLOAT_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeFloatArray(val);
+        }
+    }
+
+    /**
+     * @param val Double array.
+     */
+    void doWriteDoubleArray(@Nullable double[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DOUBLE_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeDoubleArray(val);
+        }
+    }
+
+    /**
+     * @param val Char array.
+     */
+    void doWriteCharArray(@Nullable char[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(CHAR_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeCharArray(val);
+        }
+    }
+
+    /**
+     * @param val Boolean array.
+     */
+    void doWriteBooleanArray(@Nullable boolean[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(BOOLEAN_ARR);
+            doWriteInt(val.length);
+
+            wCtx.out.writeBooleanArray(val);
+        }
+    }
+
+    /**
+     * @param val Array of strings.
+     */
+    void doWriteDecimalArray(@Nullable BigDecimal[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DECIMAL_ARR);
+            doWriteInt(val.length);
+
+            for (BigDecimal str : val)
+                doWriteDecimal(str);
+        }
+    }
+
+    /**
+     * @param val Array of strings.
+     */
+    void doWriteStringArray(@Nullable String[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(STRING_ARR);
+            doWriteInt(val.length);
+
+            for (String str : val)
+                doWriteString(str);
+        }
+    }
+
+    /**
+     * @param val Array of UUIDs.
+     */
+    void doWriteUuidArray(@Nullable UUID[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(UUID_ARR);
+            doWriteInt(val.length);
+
+            for (UUID uuid : val)
+                doWriteUuid(uuid);
+        }
+    }
+
+    /**
+     * @param val Array of dates.
+     */
+    void doWriteDateArray(@Nullable Date[] val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DATE_ARR);
+            doWriteInt(val.length);
+
+            for (Date date : val)
+                doWriteDate(date);
+        }
+    }
+
+    /**
+     * @param val Array of objects.
+     * @throws PortableException In case of error.
+     */
+    void doWriteObjectArray(@Nullable Object[] val) throws PortableException {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            PortableContext.Type type = ctx.typeId(val.getClass().getComponentType());
+
+            doWriteByte(OBJ_ARR);
+
+            if (type.registered())
+                doWriteInt(type.id());
+            else {
+                doWriteInt(UNREGISTERED_TYPE_ID);
+                doWriteString(val.getClass().getComponentType().getName());
+            }
+
+            doWriteInt(val.length);
+
+            for (Object obj : val)
+                doWriteObject(obj, false);
+        }
+    }
+
+    /**
+     * @param col Collection.
+     * @throws PortableException In case of error.
+     */
+    void doWriteCollection(@Nullable Collection<?> col) throws PortableException {
+        if (col == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(COL);
+            doWriteInt(col.size());
+            doWriteByte(ctx.collectionType(col.getClass()));
+
+            for (Object obj : col)
+                doWriteObject(obj, false);
+        }
+    }
+
+    /**
+     * @param map Map.
+     * @throws PortableException In case of error.
+     */
+    void doWriteMap(@Nullable Map<?, ?> map) throws PortableException {
+        if (map == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(MAP);
+            doWriteInt(map.size());
+            doWriteByte(ctx.mapType(map.getClass()));
+
+            for (Map.Entry<?, ?> e : map.entrySet()) {
+                doWriteObject(e.getKey(), false);
+                doWriteObject(e.getValue(), false);
+            }
+        }
+    }
+
+    /**
+     * @param e Map entry.
+     * @throws PortableException In case of error.
+     */
+    void doWriteMapEntry(@Nullable Map.Entry<?, ?> e) throws PortableException {
+        if (e == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(MAP_ENTRY);
+            doWriteObject(e.getKey(), false);
+            doWriteObject(e.getValue(), false);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void doWriteEnum(@Nullable Enum<?> val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            PortableContext.Type type = ctx.typeId(val.getClass());
+
+            doWriteByte(ENUM);
+
+            if (type.registered())
+                doWriteInt(type.id());
+            else {
+                doWriteInt(UNREGISTERED_TYPE_ID);
+                doWriteString(val.getClass().getName());
+            }
+
+            doWriteInt(val.ordinal());
+        }
+    }
+
+    /**
+     * @param val Array.
+     */
+    void doWriteEnumArray(@Nullable Object[] val) {
+        assert val == null || val.getClass().getComponentType().isEnum();
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            PortableContext.Type type = ctx.typeId(val.getClass().getComponentType());
+
+            doWriteByte(ENUM_ARR);
+
+            if (type.registered())
+                doWriteInt(type.id());
+            else {
+                doWriteInt(UNREGISTERED_TYPE_ID);
+                doWriteString(val.getClass().getComponentType().getName());
+            }
+
+            doWriteInt(val.length);
+
+            // TODO: Denis: Redundant data for each element of the array.
+            for (Object o : val)
+                doWriteEnum((Enum<?>)o);
+        }
+    }
+
+    /**
+     * @param val Class.
+     */
+    void doWriteClass(@Nullable Class val) {
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            PortableContext.Type type = ctx.typeId(val);
+
+            doWriteByte(CLASS);
+
+            if (type.registered())
+                doWriteInt(type.id());
+            else {
+                doWriteInt(UNREGISTERED_TYPE_ID);
+                doWriteString(val.getClass().getName());
+            }
+        }
+    }
+
+    /**
+     * @param po Portable object.
+     */
+    void doWritePortableObject(@Nullable PortableObjectImpl po) {
+        if (po == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(PORTABLE_OBJ);
+
+            byte[] poArr = po.array();
+
+            doWriteInt(poArr.length);
+
+            wCtx.out.writeByteArray(poArr);
+
+            doWriteInt(po.start());
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeByteField(@Nullable Byte val) {
+        doWriteInt(val != null ? 2 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(BYTE);
+            doWriteByte(val);
+        }
+    }
+
+    /**
+     * @param val Class.
+     */
+    void writeClassField(@Nullable Class val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteClass(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeShortField(@Nullable Short val) {
+        doWriteInt(val != null ? 3 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(SHORT);
+            doWriteShort(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeIntField(@Nullable Integer val) {
+        doWriteInt(val != null ? 5 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(INT);
+            doWriteInt(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeLongField(@Nullable Long val) {
+        doWriteInt(val != null ? 9 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(LONG);
+            doWriteLong(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeFloatField(@Nullable Float val) {
+        doWriteInt(val != null ? 5 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(FLOAT);
+            doWriteFloat(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDoubleField(@Nullable Double val) {
+        doWriteInt(val != null ? 9 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(DOUBLE);
+            doWriteDouble(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeCharField(@Nullable Character val) {
+        doWriteInt(val != null ? 3 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(CHAR);
+            doWriteChar(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeBooleanField(@Nullable Boolean val) {
+        doWriteInt(val != null ? 2 : 1);
+
+        if (val == null)
+            doWriteByte(NULL);
+        else {
+            doWriteByte(BOOLEAN);
+            doWriteBoolean(val);
+        }
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDecimalField(@Nullable BigDecimal val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteDecimal(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeStringField(@Nullable String val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteString(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeUuidField(@Nullable UUID val) {
+        doWriteInt(val != null ? 17 : 1);
+        doWriteUuid(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDateField(@Nullable Date val) {
+        doWriteInt(val != null ? 13 : 1);
+        doWriteDate(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeTimestampField(@Nullable Timestamp val) {
+        doWriteInt(val != null ? 13 : 1);
+        doWriteTimestamp(val);
+    }
+
+    /**
+     * @param obj Object.
+     * @throws PortableException In case of error.
+     */
+    void writeObjectField(@Nullable Object obj) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWriteObject(obj, false);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeByteArrayField(@Nullable byte[] val) {
+        doWriteInt(val != null ? 5 + val.length : 1);
+        doWriteByteArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeShortArrayField(@Nullable short[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 1) : 1);
+        doWriteShortArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeIntArrayField(@Nullable int[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 2) : 1);
+        doWriteIntArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeLongArrayField(@Nullable long[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 3) : 1);
+        doWriteLongArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeFloatArrayField(@Nullable float[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 2) : 1);
+        doWriteFloatArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDoubleArrayField(@Nullable double[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 3) : 1);
+        doWriteDoubleArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeCharArrayField(@Nullable char[] val) {
+        doWriteInt(val != null ? 5 + (val.length << 1) : 1);
+        doWriteCharArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeBooleanArrayField(@Nullable boolean[] val) {
+        doWriteInt(val != null ? 5 + val.length : 1);
+        doWriteBooleanArray(val);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDecimalArrayField(@Nullable BigDecimal[] val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteDecimalArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeStringArrayField(@Nullable String[] val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteStringArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeUuidArrayField(@Nullable UUID[] val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteUuidArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeDateArrayField(@Nullable Date[] val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteDateArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     * @throws PortableException In case of error.
+     */
+    void writeObjectArrayField(@Nullable Object[] val) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWriteObjectArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param col Collection.
+     * @throws PortableException In case of error.
+     */
+    void writeCollectionField(@Nullable Collection<?> col) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWriteCollection(col);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param map Map.
+     * @throws PortableException In case of error.
+     */
+    void writeMapField(@Nullable Map<?, ?> map) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWriteMap(map);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param e Map entry.
+     * @throws PortableException In case of error.
+     */
+    void writeMapEntryField(@Nullable Map.Entry<?, ?> e) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWriteMapEntry(e);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeEnumField(@Nullable Enum<?> val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteEnum(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param val Value.
+     */
+    void writeEnumArrayField(@Nullable Object[] val) {
+        int lenPos = reserveAndMark(4);
+
+        doWriteEnumArray(val);
+
+        writeDelta(lenPos);
+    }
+
+    /**
+     * @param po Portable object.
+     * @throws PortableException In case of error.
+     */
+    void writePortableObjectField(@Nullable PortableObjectImpl po) throws PortableException {
+        int lenPos = reserveAndMark(4);
+
+        doWritePortableObject(po);
+
+        writeDelta(lenPos);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByte(String fieldName, byte val) throws PortableException {
+        writeFieldId(fieldName, BYTE);
+        writeByteField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByte(byte val) throws PortableException {
+        doWriteByte(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShort(String fieldName, short val) throws PortableException {
+        writeFieldId(fieldName, SHORT);
+        writeShortField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShort(short val) throws PortableException {
+        doWriteShort(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeInt(String fieldName, int val) throws PortableException {
+        writeFieldId(fieldName, INT);
+        writeIntField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeInt(int val) throws PortableException {
+        doWriteInt(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLong(String fieldName, long val) throws PortableException {
+        writeFieldId(fieldName, LONG);
+        writeLongField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLong(long val) throws PortableException {
+        doWriteLong(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloat(String fieldName, float val) throws PortableException {
+        writeFieldId(fieldName, FLOAT);
+        writeFloatField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloat(float val) throws PortableException {
+        doWriteFloat(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDouble(String fieldName, double val) throws PortableException {
+        writeFieldId(fieldName, DOUBLE);
+        writeDoubleField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDouble(double val) throws PortableException {
+        doWriteDouble(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeChar(String fieldName, char val) throws PortableException {
+        writeFieldId(fieldName, CHAR);
+        writeCharField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeChar(char val) throws PortableException {
+        doWriteChar(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBoolean(String fieldName, boolean val) throws PortableException {
+        writeFieldId(fieldName, BOOLEAN);
+        writeBooleanField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBoolean(boolean val) throws PortableException {
+        doWriteBoolean(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDecimal(String fieldName, @Nullable BigDecimal val) throws PortableException {
+        writeFieldId(fieldName, DECIMAL);
+        writeDecimalField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDecimal(@Nullable BigDecimal val) throws PortableException {
+        doWriteDecimal(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeString(String fieldName, @Nullable String val) throws PortableException {
+        writeFieldId(fieldName, STRING);
+        writeStringField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeString(@Nullable String val) throws PortableException {
+        doWriteString(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeUuid(String fieldName, @Nullable UUID val) throws PortableException {
+        writeFieldId(fieldName, UUID);
+        writeUuidField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeUuid(@Nullable UUID val) throws PortableException {
+        doWriteUuid(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDate(String fieldName, @Nullable Date val) throws PortableException {
+        writeFieldId(fieldName, DATE);
+        writeDateField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDate(@Nullable Date val) throws PortableException {
+        doWriteDate(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeTimestamp(String fieldName, @Nullable Timestamp val) throws PortableException {
+        writeFieldId(fieldName, DATE);
+        writeTimestampField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeTimestamp(@Nullable Timestamp val) throws PortableException {
+        doWriteTimestamp(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeObject(String fieldName, @Nullable Object obj) throws PortableException {
+        writeFieldId(fieldName, OBJ);
+        writeObjectField(obj);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeObject(@Nullable Object obj) throws PortableException {
+        doWriteObject(obj, false);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeObjectDetached(@Nullable Object obj) throws PortableException {
+        doWriteObject(obj, true);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByteArray(String fieldName, @Nullable byte[] val) throws PortableException {
+        writeFieldId(fieldName, BYTE_ARR);
+        writeByteArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByteArray(@Nullable byte[] val) throws PortableException {
+        doWriteByteArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShortArray(String fieldName, @Nullable short[] val) throws PortableException {
+        writeFieldId(fieldName, SHORT_ARR);
+        writeShortArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShortArray(@Nullable short[] val) throws PortableException {
+        doWriteShortArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeIntArray(String fieldName, @Nullable int[] val) throws PortableException {
+        writeFieldId(fieldName, INT_ARR);
+        writeIntArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeIntArray(@Nullable int[] val) throws PortableException {
+        doWriteIntArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLongArray(String fieldName, @Nullable long[] val) throws PortableException {
+        writeFieldId(fieldName, LONG_ARR);
+        writeLongArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLongArray(@Nullable long[] val) throws PortableException {
+        doWriteLongArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloatArray(String fieldName, @Nullable float[] val) throws PortableException {
+        writeFieldId(fieldName, FLOAT_ARR);
+        writeFloatArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloatArray(@Nullable float[] val) throws PortableException {
+        doWriteFloatArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDoubleArray(String fieldName, @Nullable double[] val)
+        throws PortableException {
+        writeFieldId(fieldName, DOUBLE_ARR);
+        writeDoubleArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDoubleArray(@Nullable double[] val) throws PortableException {
+        doWriteDoubleArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeCharArray(String fieldName, @Nullable char[] val) throws PortableException {
+        writeFieldId(fieldName, CHAR_ARR);
+        writeCharArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeCharArray(@Nullable char[] val) throws PortableException {
+        doWriteCharArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBooleanArray(String fieldName, @Nullable boolean[] val)
+        throws PortableException {
+        writeFieldId(fieldName, BOOLEAN_ARR);
+        writeBooleanArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBooleanArray(@Nullable boolean[] val) throws PortableException {
+        doWriteBooleanArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDecimalArray(String fieldName, @Nullable BigDecimal[] val)
+        throws PortableException {
+        writeFieldId(fieldName, DECIMAL_ARR);
+        writeDecimalArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDecimalArray(@Nullable BigDecimal[] val) throws PortableException {
+        doWriteDecimalArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeStringArray(String fieldName, @Nullable String[] val)
+        throws PortableException {
+        writeFieldId(fieldName, STRING_ARR);
+        writeStringArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeStringArray(@Nullable String[] val) throws PortableException {
+        doWriteStringArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeUuidArray(String fieldName, @Nullable UUID[] val) throws PortableException {
+        writeFieldId(fieldName, UUID_ARR);
+        writeUuidArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeUuidArray(@Nullable UUID[] val) throws PortableException {
+        doWriteUuidArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDateArray(String fieldName, @Nullable Date[] val) throws PortableException {
+        writeFieldId(fieldName, DATE_ARR);
+        writeDateArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDateArray(@Nullable Date[] val) throws PortableException {
+        doWriteDateArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeObjectArray(String fieldName, @Nullable Object[] val) throws PortableException {
+        writeFieldId(fieldName, OBJ_ARR);
+        writeObjectArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeObjectArray(@Nullable Object[] val) throws PortableException {
+        doWriteObjectArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T> void writeCollection(String fieldName, @Nullable Collection<T> col)
+        throws PortableException {
+        writeFieldId(fieldName, COL);
+        writeCollectionField(col);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T> void writeCollection(@Nullable Collection<T> col) throws PortableException {
+        doWriteCollection(col);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <K, V> void writeMap(String fieldName, @Nullable Map<K, V> map)
+        throws PortableException {
+        writeFieldId(fieldName, MAP);
+        writeMapField(map);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <K, V> void writeMap(@Nullable Map<K, V> map) throws PortableException {
+        doWriteMap(map);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T extends Enum<?>> void writeEnum(String fieldName, T val) throws PortableException {
+        writeFieldId(fieldName, ENUM);
+        writeEnumField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T extends Enum<?>> void writeEnum(T val) throws PortableException {
+        doWriteEnum(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T extends Enum<?>> void writeEnumArray(String fieldName, T[] val) throws PortableException {
+        writeFieldId(fieldName, ENUM_ARR);
+        writeEnumArrayField(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public <T extends Enum<?>> void writeEnumArray(T[] val) throws PortableException {
+        doWriteEnumArray(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public PortableRawWriter rawWriter() {
+        if (allowFields) {
+            wCtx.out.writeInt(start + RAW_DATA_OFF_POS, wCtx.out.position() - start);
+
+            allowFields = false;
+        }
+
+        return this;
+    }
+
+    /** {@inheritDoc} */
+    @Override public PortableOutputStream out() {
+        return wCtx.out;
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBytes(String s) throws IOException {
+        int len = s.length();
+
+        writeInt(len);
+
+        for (int i = 0; i < len; i++)
+            writeByte(s.charAt(i));
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeChars(String s) throws IOException {
+        int len = s.length();
+
+        writeInt(len);
+
+        for (int i = 0; i < len; i++)
+            writeChar(s.charAt(i));
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeUTF(String s) throws IOException {
+        writeString(s);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByte(int v) throws IOException {
+        doWriteByte((byte)v);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShort(int v) throws IOException {
+        doWriteShort((short)v);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeChar(int v) throws IOException {
+        doWriteChar((char)v);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void write(int b) throws IOException {
+        doWriteByte((byte)b);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void flush() throws IOException {
+        // No-op.
+    }
+
+    /**
+     * Reserve a room for an integer.
+     *
+     * @return Position in the stream where value is to be written.
+     */
+    public int reserveInt() {
+        return reserve(LEN_INT);
+    }
+
+    /**
+     * Write int value at the specific position.
+     *
+     * @param pos Position.
+     * @param val Value.
+     * @throws PortableException If failed.
+     */
+    public void writeInt(int pos, int val) throws PortableException {
+        wCtx.out.writeInt(pos, val);
+    }
+
+    /**
+     * @param fieldName Field name.
+     * @throws PortableException If fields are not allowed.
+     */
+    private void writeFieldId(String fieldName, byte fieldType) throws PortableException {
+        A.notNull(fieldName, "fieldName");
+
+        if (!allowFields)
+            throw new PortableException("Individual field can't be written after raw writer is acquired " +
+                "via rawWriter() method. Consider fixing serialization logic for class: " + cls.getName());
+
+        int id = ctx.fieldId(typeId, fieldName);
+
+        if (metaEnabled)
+            metaHashSum = 31 * metaHashSum + (id + fieldType);
+
+        doWriteInt(id);
+    }
+
+    /**
+     * Create new writer with same context.
+     * @param typeId type
+     * @return New writer.
+     */
+    PortableWriterExImpl newWriter(int typeId) {
+        PortableWriterExImpl res = new PortableWriterExImpl(ctx, wCtx);
+
+        res.typeId = typeId;
+
+        return res;
+    }
+
+    /**
+     * @return Portable context.
+     */
+    PortableContext context() {
+        return ctx;
+    }
+
+    /** */
+    private static class WriterContext {
+        /** */
+        private Map<Object, Integer> handles = new IdentityHashMap<>();
+
+        /** Output stream. */
+        private PortableOutputStream out;
+
+        /**
+         * Constructor.
+         *
+         * @param out Output stream.
+         * @param handles Handles.
+         */
+        private WriterContext(PortableOutputStream out, Map<Object, Integer> handles) {
+            this.out = out;
+            this.handles = handles == null ? new IdentityHashMap<Object, Integer>() : handles;
+        }
+
+        /**
+         * @param obj Object.
+         * @return Handle.
+         */
+        private int handle(Object obj) {
+            assert obj != null;
+
+            Integer h = handles.get(obj);
+
+            if (h != null)
+                return out.position() - h;
+            else {
+                handles.put(obj, out.position());
+
+                return -1;
+            }
+        }
+
+        /**
+         *
+         */
+        private void resetHandles() {
+            handles = new IdentityHashMap<>();
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/package-info.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/package-info.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/package-info.java
new file mode 100644
index 0000000..ccf9fad
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/package-info.java
@@ -0,0 +1,22 @@
+/*
+ * 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 description. -->
+ * Contains portable APIs internal implementation.
+ */
+package org.apache.ignite.internal.portable;
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractInputStream.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractInputStream.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractInputStream.java
new file mode 100644
index 0000000..9f0b1db
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractInputStream.java
@@ -0,0 +1,343 @@
+/*
+ * 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.portable.streams;
+
+import org.apache.ignite.portable.*;
+
+/**
+ * Portable abstract input stream.
+ */
+public abstract class PortableAbstractInputStream extends PortableAbstractStream
+    implements PortableInputStream {
+    /** Length of data inside array. */
+    protected int len;
+
+    /** {@inheritDoc} */
+    @Override public byte readByte() {
+        ensureEnoughData(1);
+
+        return readByteAndShift();
+    }
+
+    /** {@inheritDoc} */
+    @Override public byte[] readByteArray(int cnt) {
+        ensureEnoughData(cnt);
+
+        byte[] res = new byte[cnt];
+
+        copyAndShift(res, BYTE_ARR_OFF, cnt);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public boolean readBoolean() {
+        return readByte() == BYTE_ONE;
+    }
+
+    /** {@inheritDoc} */
+    @Override public boolean[] readBooleanArray(int cnt) {
+        ensureEnoughData(cnt);
+
+        boolean[] res = new boolean[cnt];
+
+        copyAndShift(res, BOOLEAN_ARR_OFF, cnt);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public short readShort() {
+        ensureEnoughData(2);
+
+        short res = readShortFast();
+
+        shift(2);
+
+        if (!LITTLE_ENDIAN)
+            res = Short.reverseBytes(res);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public short[] readShortArray(int cnt) {
+        int len = cnt << 1;
+
+        ensureEnoughData(len);
+
+        short[] res = new short[cnt];
+
+        copyAndShift(res, SHORT_ARR_OFF, len);
+
+        if (!LITTLE_ENDIAN) {
+            for (int i = 0; i < res.length; i++)
+                res[i] = Short.reverseBytes(res[i]);
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public char readChar() {
+        ensureEnoughData(2);
+
+        char res = readCharFast();
+
+        shift(2);
+
+        if (!LITTLE_ENDIAN)
+            res = Character.reverseBytes(res);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public char[] readCharArray(int cnt) {
+        int len = cnt << 1;
+
+        ensureEnoughData(len);
+
+        char[] res = new char[cnt];
+
+        copyAndShift(res, CHAR_ARR_OFF, len);
+
+        if (!LITTLE_ENDIAN) {
+            for (int i = 0; i < res.length; i++)
+                res[i] = Character.reverseBytes(res[i]);
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public int readInt() {
+        ensureEnoughData(4);
+
+        int res = readIntFast();
+
+        shift(4);
+
+        if (!LITTLE_ENDIAN)
+            res = Integer.reverseBytes(res);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public int[] readIntArray(int cnt) {
+        int len = cnt << 2;
+
+        ensureEnoughData(len);
+
+        int[] res = new int[cnt];
+
+        copyAndShift(res, INT_ARR_OFF, len);
+
+        if (!LITTLE_ENDIAN) {
+            for (int i = 0; i < res.length; i++)
+                res[i] = Integer.reverseBytes(res[i]);
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public int readInt(int pos) {
+        int delta = pos + 4 - this.pos;
+
+        if (delta > 0)
+            ensureEnoughData(delta);
+
+        return readIntPositioned(pos);
+    }
+
+    /** {@inheritDoc} */
+    @Override public float readFloat() {
+        return Float.intBitsToFloat(readInt());
+    }
+
+    /** {@inheritDoc} */
+    @Override public float[] readFloatArray(int cnt) {
+        int len = cnt << 2;
+
+        ensureEnoughData(len);
+
+        float[] res = new float[cnt];
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(res, FLOAT_ARR_OFF, len);
+        else {
+            for (int i = 0; i < res.length; i++) {
+                int x = readIntFast();
+
+                shift(4);
+
+                res[i] = Float.intBitsToFloat(Integer.reverseBytes(x));
+            }
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public long readLong() {
+        ensureEnoughData(8);
+
+        long res = readLongFast();
+
+        shift(8);
+
+        if (!LITTLE_ENDIAN)
+            res = Long.reverseBytes(res);
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public long[] readLongArray(int cnt) {
+        int len = cnt << 3;
+
+        ensureEnoughData(len);
+
+        long[] res = new long[cnt];
+
+        copyAndShift(res, LONG_ARR_OFF, len);
+
+        if (!LITTLE_ENDIAN) {
+            for (int i = 0; i < res.length; i++)
+                res[i] = Long.reverseBytes(res[i]);
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public double readDouble() {
+        return Double.longBitsToDouble(readLong());
+    }
+
+    /** {@inheritDoc} */
+    @Override public double[] readDoubleArray(int cnt) {
+        int len = cnt << 3;
+
+        ensureEnoughData(len);
+
+        double[] res = new double[cnt];
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(res, DOUBLE_ARR_OFF, len);
+        else {
+            for (int i = 0; i < res.length; i++) {
+                long x = readLongFast();
+
+                shift(8);
+
+                res[i] = Double.longBitsToDouble(Long.reverseBytes(x));
+            }
+        }
+
+        return res;
+    }
+
+    /** {@inheritDoc} */
+    @Override public int read(byte[] arr, int off, int len) {
+        if (len > remaining())
+            len = remaining();
+
+        copyAndShift(arr, BYTE_ARR_OFF + off, len);
+
+        return len;
+    }
+
+    /** {@inheritDoc} */
+    @Override public void position(int pos) {
+        if (remaining() + this.pos < pos)
+            throw new PortableException("Position is out of bounds: " + pos);
+        else
+            this.pos = pos;
+    }
+
+    /** {@inheritDoc} */
+    @Override public long offheapPointer() {
+        return 0;
+    }
+
+    /**
+     * Ensure that there is enough data.
+     *
+     * @param cnt Length.
+     */
+    protected void ensureEnoughData(int cnt) {
+        if (remaining() < cnt)
+            throw new PortableException("Not enough data to read the value [position=" + pos +
+                ", requiredBytes=" + cnt + ", remainingBytes=" + remaining() + ']');
+    }
+
+    /**
+     * Read next byte from the stream and perform shift.
+     *
+     * @return Next byte.
+     */
+    protected abstract byte readByteAndShift();
+
+    /**
+     * Copy data to target object shift position afterwards.
+     *
+     * @param target Target.
+     * @param off Offset.
+     * @param len Length.
+     */
+    protected abstract void copyAndShift(Object target, long off, int len);
+
+    /**
+     * Read short value (fast path).
+     *
+     * @return Short value.
+     */
+    protected abstract short readShortFast();
+
+    /**
+     * Read char value (fast path).
+     *
+     * @return Char value.
+     */
+    protected abstract char readCharFast();
+
+    /**
+     * Read int value (fast path).
+     *
+     * @return Int value.
+     */
+    protected abstract int readIntFast();
+
+    /**
+     * Read long value (fast path).
+     *
+     * @return Long value.
+     */
+    protected abstract long readLongFast();
+
+    /**
+     * Internal routine for positioned int value read.
+     *
+     * @param pos Position.
+     * @return Int value.
+     */
+    protected abstract int readIntPositioned(int pos);
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractOutputStream.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractOutputStream.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractOutputStream.java
new file mode 100644
index 0000000..2c3179a
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractOutputStream.java
@@ -0,0 +1,323 @@
+/*
+ * 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.portable.streams;
+
+/**
+ * Base portable output stream.
+ */
+public abstract class PortableAbstractOutputStream extends PortableAbstractStream
+    implements PortableOutputStream {
+    /** Minimal capacity when it is reasonable to start doubling resize. */
+    private static final int MIN_CAP = 256;
+
+    /** {@inheritDoc} */
+    @Override public void writeByte(byte val) {
+        ensureCapacity(pos + 1);
+
+        writeByteAndShift(val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeByteArray(byte[] val) {
+        ensureCapacity(pos + val.length);
+
+        copyAndShift(val, BYTE_ARR_OFF, val.length);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBoolean(boolean val) {
+        writeByte(val ? BYTE_ONE : BYTE_ZERO);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeBooleanArray(boolean[] val) {
+        ensureCapacity(pos + val.length);
+
+        copyAndShift(val, BOOLEAN_ARR_OFF, val.length);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShort(short val) {
+        ensureCapacity(pos + 2);
+
+        if (!LITTLE_ENDIAN)
+            val = Short.reverseBytes(val);
+
+        writeShortFast(val);
+
+        shift(2);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeShortArray(short[] val) {
+        int cnt = val.length << 1;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, SHORT_ARR_OFF, cnt);
+        else {
+            for (short item : val)
+                writeShortFast(Short.reverseBytes(item));
+
+            shift(cnt);
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeChar(char val) {
+        ensureCapacity(pos + 2);
+
+        if (!LITTLE_ENDIAN)
+            val = Character.reverseBytes(val);
+
+        writeCharFast(val);
+
+        shift(2);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeCharArray(char[] val) {
+        int cnt = val.length << 1;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, CHAR_ARR_OFF, cnt);
+        else {
+            for (char item : val)
+                writeCharFast(Character.reverseBytes(item));
+
+            shift(cnt);
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeInt(int val) {
+        ensureCapacity(pos + 4);
+
+        if (!LITTLE_ENDIAN)
+            val = Integer.reverseBytes(val);
+
+        writeIntFast(val);
+
+        shift(4);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeInt(int pos, int val) {
+        ensureCapacity(pos + 4);
+
+        writeIntPositioned(pos, val);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeIntArray(int[] val) {
+        int cnt = val.length << 2;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, INT_ARR_OFF, cnt);
+        else {
+            for (int item : val)
+                writeIntFast(Integer.reverseBytes(item));
+
+            shift(cnt);
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloat(float val) {
+        writeInt(Float.floatToIntBits(val));
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeFloatArray(float[] val) {
+        int cnt = val.length << 2;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, FLOAT_ARR_OFF, cnt);
+        else {
+            for (float item : val) {
+                writeIntFast(Integer.reverseBytes(Float.floatToIntBits(item)));
+
+                shift(4);
+            }
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLong(long val) {
+        ensureCapacity(pos + 8);
+
+        if (!LITTLE_ENDIAN)
+            val = Long.reverseBytes(val);
+
+        writeLongFast(val);
+
+        shift(8);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeLongArray(long[] val) {
+        int cnt = val.length << 3;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, LONG_ARR_OFF, cnt);
+        else {
+            for (long item : val)
+                writeLongFast(Long.reverseBytes(item));
+
+            shift(cnt);
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDouble(double val) {
+        writeLong(Double.doubleToLongBits(val));
+    }
+
+    /** {@inheritDoc} */
+    @Override public void writeDoubleArray(double[] val) {
+        int cnt = val.length << 3;
+
+        ensureCapacity(pos + cnt);
+
+        if (LITTLE_ENDIAN)
+            copyAndShift(val, DOUBLE_ARR_OFF, cnt);
+        else {
+            for (double item : val) {
+                writeLongFast(Long.reverseBytes(Double.doubleToLongBits(item)));
+
+                shift(8);
+            }
+        }
+    }
+
+    /** {@inheritDoc} */
+    @Override public void write(byte[] arr, int off, int len) {
+        ensureCapacity(pos + len);
+
+        copyAndShift(arr, BYTE_ARR_OFF + off, len);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void write(long addr, int cnt) {
+        ensureCapacity(pos + cnt);
+
+        copyAndShift(null, addr, cnt);
+    }
+
+    /** {@inheritDoc} */
+    @Override public void position(int pos) {
+        ensureCapacity(pos);
+
+        this.pos = pos;
+    }
+
+    /** {@inheritDoc} */
+    @Override public long offheapPointer() {
+        return 0;
+    }
+
+    /**
+     * Calculate new capacity.
+     *
+     * @param curCap Current capacity.
+     * @param reqCap Required capacity.
+     * @return New capacity.
+     */
+    protected static int capacity(int curCap, int reqCap) {
+        int newCap;
+
+        if (reqCap < MIN_CAP)
+            newCap = MIN_CAP;
+        else {
+            newCap = curCap << 1;
+
+            if (newCap < reqCap)
+                newCap = reqCap;
+        }
+
+        return newCap;
+    }
+
+    /**
+     * Write next byte to the stream.
+     *
+     * @param val Value.
+     */
+    protected abstract void writeByteAndShift(byte val);
+
+    /**
+     * Copy source object to the stream shift position afterwards.
+     *
+     * @param src Source.
+     * @param off Offset.
+     * @param len Length.
+     */
+    protected abstract void copyAndShift(Object src, long off, int len);
+
+    /**
+     * Write short value (fast path).
+     *
+     * @param val Short value.
+     */
+    protected abstract void writeShortFast(short val);
+
+    /**
+     * Write char value (fast path).
+     *
+     * @param val Char value.
+     */
+    protected abstract void writeCharFast(char val);
+
+    /**
+     * Write int value (fast path).
+     *
+     * @param val Int value.
+     */
+    protected abstract void writeIntFast(int val);
+
+    /**
+     * Write long value (fast path).
+     *
+     * @param val Long value.
+     */
+    protected abstract void writeLongFast(long val);
+
+    /**
+     * Write int value to the given position.
+     *
+     * @param pos Position.
+     * @param val Value.
+     */
+    protected abstract void writeIntPositioned(int pos, int val);
+
+    /**
+     * Ensure capacity.
+     *
+     * @param cnt Required byte count.
+     */
+    protected abstract void ensureCapacity(int cnt);
+}

http://git-wip-us.apache.org/repos/asf/ignite/blob/878dcd92/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractStream.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractStream.java b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractStream.java
new file mode 100644
index 0000000..a39662b
--- /dev/null
+++ b/modules/core/src/main/java/org/apache/ignite/internal/portable/streams/PortableAbstractStream.java
@@ -0,0 +1,82 @@
+/*
+ * 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.portable.streams;
+
+import org.apache.ignite.internal.util.*;
+
+import sun.misc.*;
+
+import java.nio.*;
+
+/**
+ * Portable abstract stream.
+ */
+public abstract class PortableAbstractStream implements PortableStream {
+    /** Byte: zero. */
+    protected static final byte BYTE_ZERO = 0;
+
+    /** Byte: one. */
+    protected static final byte BYTE_ONE = 1;
+
+    /** Whether little endian is used on the platform. */
+    protected static final boolean LITTLE_ENDIAN = ByteOrder.nativeOrder() == ByteOrder.LITTLE_ENDIAN;
+
+    /** Unsafe instance. */
+    protected static final Unsafe UNSAFE = GridUnsafe.unsafe();
+
+    /** Array offset: boolean. */
+    protected static final long BOOLEAN_ARR_OFF = UNSAFE.arrayBaseOffset(boolean[].class);
+
+    /** Array offset: byte. */
+    protected static final long BYTE_ARR_OFF = UNSAFE.arrayBaseOffset(byte[].class);
+
+    /** Array offset: short. */
+    protected static final long SHORT_ARR_OFF = UNSAFE.arrayBaseOffset(short[].class);
+
+    /** Array offset: char. */
+    protected static final long CHAR_ARR_OFF = UNSAFE.arrayBaseOffset(char[].class);
+
+    /** Array offset: int. */
+    protected static final long INT_ARR_OFF = UNSAFE.arrayBaseOffset(int[].class);
+
+    /** Array offset: float. */
+    protected static final long FLOAT_ARR_OFF = UNSAFE.arrayBaseOffset(float[].class);
+
+    /** Array offset: long. */
+    protected static final long LONG_ARR_OFF = UNSAFE.arrayBaseOffset(long[].class);
+
+    /** Array offset: double. */
+    protected static final long DOUBLE_ARR_OFF = UNSAFE.arrayBaseOffset(double[].class);
+
+    /** Position. */
+    protected int pos;
+
+    /** {@inheritDoc} */
+    @Override public int position() {
+        return pos;
+    }
+
+    /**
+     * Shift position.
+     *
+     * @param cnt Byte count.
+     */
+    protected void shift(int cnt) {
+        pos += cnt;
+    }
+}