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 2017/03/15 14:37:21 UTC

[3/3] ignite git commit: ignite-4768 txs

ignite-4768 txs


Project: http://git-wip-us.apache.org/repos/asf/ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/ignite/commit/2272fadc
Tree: http://git-wip-us.apache.org/repos/asf/ignite/tree/2272fadc
Diff: http://git-wip-us.apache.org/repos/asf/ignite/diff/2272fadc

Branch: refs/heads/ignite-4768-1
Commit: 2272fadc5989b2787f14454c91eecde9b1fa7727
Parents: 684d66c
Author: sboikov <sb...@gridgain.com>
Authored: Wed Mar 15 11:01:45 2017 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Mar 15 17:37:02 2017 +0300

----------------------------------------------------------------------
 .../ClientAbstractMultiNodeSelfTest.java        |  13 +-
 .../processors/cache/GridCacheAdapter.java      |  10 +-
 .../processors/cache/GridCacheIoManager.java    |   3 +
 .../cache/GridCacheSharedContext.java           |   8 +-
 .../processors/cache/GridCacheUtils.java        |   3 +-
 .../distributed/GridCacheCommittedTxInfo.java   | 117 -----
 .../GridDistributedTxRemoteAdapter.java         |  55 +-
 .../dht/GridDhtTransactionalCacheAdapter.java   |  18 +-
 .../cache/distributed/dht/GridDhtTxLocal.java   | 122 ++---
 .../distributed/dht/GridDhtTxLocalAdapter.java  |  12 +-
 .../distributed/dht/GridDhtTxPrepareFuture.java |  69 ++-
 .../colocated/GridDhtColocatedLockFuture.java   |   7 +-
 .../distributed/near/GridNearLockFuture.java    |   4 +-
 .../distributed/near/GridNearLockRequest.java   | 200 ++------
 .../near/GridNearOptimisticTxPrepareFuture.java |   2 +-
 .../near/GridNearTxFinishFuture.java            |   4 +-
 .../cache/distributed/near/GridNearTxLocal.java |  90 +++-
 .../near/GridNearTxPrepareFutureAdapter.java    |   5 +-
 .../near/GridNearTxPrepareRequest.java          |   2 +-
 .../distributed/near/GridNearTxRemote.java      |   4 +-
 .../cache/transactions/IgniteInternalTx.java    |  68 +--
 .../cache/transactions/IgniteTxAdapter.java     | 136 +----
 .../cache/transactions/IgniteTxHandler.java     |  27 +-
 .../IgniteTxImplicitSingleStateImpl.java        |   4 +-
 .../transactions/IgniteTxLocalAdapter.java      |  36 +-
 .../cache/transactions/IgniteTxLocalEx.java     |   9 +-
 .../cache/transactions/IgniteTxManager.java     | 148 ++----
 .../IgniteTxRemoteStateAdapter.java             |   2 +-
 .../cache/transactions/IgniteTxState.java       |   2 +-
 .../cache/transactions/IgniteTxStateImpl.java   |   4 +-
 .../datastructures/DataStructuresProcessor.java |  28 +-
 .../datastructures/GridCacheAtomicLongImpl.java |  16 +-
 .../GridCacheAtomicReferenceImpl.java           |   4 +-
 .../GridCacheAtomicSequenceImpl.java            |   2 +-
 .../GridCacheAtomicStampedImpl.java             |   4 +-
 .../GridCacheCountDownLatchImpl.java            |   4 +-
 .../datastructures/GridCacheLockImpl.java       |  11 +-
 .../datastructures/GridCacheSemaphoreImpl.java  |  11 +-
 .../GridTransactionalCacheQueueImpl.java        |  10 +-
 .../processors/igfs/IgfsDataManager.java        |   4 +-
 .../processors/igfs/IgfsMetaManager.java        |  34 +-
 .../service/GridServiceProcessor.java           |   2 +-
 .../internal/TestRecordingCommunicationSpi.java |  29 ++
 .../cache/IgniteTxConfigCacheSelfTest.java      |   2 +-
 .../IgniteCacheSystemTransactionsSelfTest.java  |   6 +-
 ...xOriginatingNodeFailureAbstractSelfTest.java |   2 +-
 ...cOriginatingNodeFailureAbstractSelfTest.java |   4 +-
 ...ePrimaryNodeFailureRecoveryAbstractTest.java |   4 +-
 .../dht/IgniteCacheTxRecoveryRollbackTest.java  | 501 +++++++++++++++++++
 .../GridCachePartitionedTxSalvageSelfTest.java  |   7 +-
 ...lockMessageSystemPoolStarvationSelfTest.java |   2 +-
 .../IgniteCacheTxRecoverySelfTestSuite.java     |   3 +
 .../processors/cache/jta/CacheJtaResource.java  |  11 +-
 53 files changed, 956 insertions(+), 929 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/clients/src/test/java/org/apache/ignite/internal/client/integration/ClientAbstractMultiNodeSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/clients/src/test/java/org/apache/ignite/internal/client/integration/ClientAbstractMultiNodeSelfTest.java b/modules/clients/src/test/java/org/apache/ignite/internal/client/integration/ClientAbstractMultiNodeSelfTest.java
index 7fb2385..2fba49a 100644
--- a/modules/clients/src/test/java/org/apache/ignite/internal/client/integration/ClientAbstractMultiNodeSelfTest.java
+++ b/modules/clients/src/test/java/org/apache/ignite/internal/client/integration/ClientAbstractMultiNodeSelfTest.java
@@ -56,6 +56,7 @@ import org.apache.ignite.internal.managers.communication.GridIoMessage;
 import org.apache.ignite.internal.processors.cache.GridCacheContext;
 import org.apache.ignite.internal.processors.cache.distributed.GridDistributedLockRequest;
 import org.apache.ignite.internal.processors.cache.transactions.IgniteInternalTx;
+import org.apache.ignite.internal.processors.cache.transactions.IgniteTxLocalAdapter;
 import org.apache.ignite.internal.processors.cache.transactions.IgniteTxManager;
 import org.apache.ignite.internal.processors.cache.version.GridCacheVersion;
 import org.apache.ignite.internal.processors.cache.version.GridCacheVersionable;
@@ -480,11 +481,11 @@ public abstract class ClientAbstractMultiNodeSelfTest extends GridCommonAbstract
     @SuppressWarnings("unchecked")
     private static class TestCommunicationSpi extends TcpCommunicationSpi {
         /** {@inheritDoc} */
-        @Override public void sendMessage(ClusterNode node, Message msg, IgniteInClosure<IgniteException> ackClosure)
+        @Override public void sendMessage(ClusterNode node, Message msg, IgniteInClosure<IgniteException> ackC)
             throws IgniteSpiException {
             checkSyncFlags((GridIoMessage)msg);
 
-            super.sendMessage(node, msg, ackClosure);
+            super.sendMessage(node, msg, ackC);
         }
 
         /**
@@ -512,13 +513,13 @@ public abstract class ClientAbstractMultiNodeSelfTest extends GridCommonAbstract
             IgniteInternalTx t = tm.tx(v);
 
             if (t.hasWriteKey(cacheCtx.txKey(cacheCtx.toCacheKeyObject("x1"))))
-                assertEquals("Invalid tx flags: " + t, FULL_ASYNC, t.syncMode());
+                assertEquals("Invalid tx flags: " + t, FULL_ASYNC, ((IgniteTxLocalAdapter)t).syncMode());
             else if (t.hasWriteKey(cacheCtx.txKey(cacheCtx.toCacheKeyObject("x2"))))
-                assertEquals("Invalid tx flags: " + t, FULL_SYNC, t.syncMode());
+                assertEquals("Invalid tx flags: " + t, FULL_SYNC, ((IgniteTxLocalAdapter)t).syncMode());
             else if (t.hasWriteKey(cacheCtx.txKey(cacheCtx.toCacheKeyObject("x3"))))
-                assertEquals("Invalid tx flags: " + t, FULL_ASYNC, t.syncMode());
+                assertEquals("Invalid tx flags: " + t, FULL_ASYNC, ((IgniteTxLocalAdapter)t).syncMode());
             else if (t.hasWriteKey(cacheCtx.txKey(cacheCtx.toCacheKeyObject("x4"))))
-                assertEquals("Invalid tx flags: " + t, FULL_SYNC, t.syncMode());
+                assertEquals("Invalid tx flags: " + t, FULL_SYNC, ((IgniteTxLocalAdapter)t).syncMode());
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
index bbf19f1..3bfd1f8 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheAdapter.java
@@ -4150,7 +4150,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                 IgniteInternalFuture<IgniteInternalTx> f = new GridEmbeddedFuture<>(fut,
                     new C2<Object, Exception, IgniteInternalFuture<IgniteInternalTx>>() {
                         @Override public IgniteInternalFuture<IgniteInternalTx> apply(Object o, Exception e) {
-                            return tx.commitTopLevelTxAsync();
+                            return tx.commitNearTxLocalAsync();
                         }
                     });
 
@@ -4159,7 +4159,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                 return f;
             }
 
-            IgniteInternalFuture<IgniteInternalTx> f = tx.commitTopLevelTxAsync();
+            IgniteInternalFuture<IgniteInternalTx> f = tx.commitNearTxLocalAsync();
 
             saveFuture(holder, f);
 
@@ -4240,7 +4240,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                 catch (IgniteCheckedException e) {
                     if (!(e instanceof IgniteTxRollbackCheckedException)) {
                         try {
-                            tx.rollbackTopLevelTx();
+                            tx.rollback();
 
                             e = new IgniteTxRollbackCheckedException("Transaction has been rolled back: " +
                                 tx.xid(), e);
@@ -4379,7 +4379,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                                         throw e;
                                     }
                                     catch (IgniteCheckedException e1) {
-                                        tx0.rollbackAsync();
+                                        tx0.rollbackNearTxLocalAsync();
 
                                         throw e1;
                                     }
@@ -4405,7 +4405,7 @@ public abstract class GridCacheAdapter<K, V> implements IgniteInternalCache<K, V
                         throw e;
                     }
                     catch (IgniteCheckedException e1) {
-                        tx0.rollbackAsync();
+                        tx0.rollbackNearTxLocalAsync();
 
                         throw e1;
                     }

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheIoManager.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheIoManager.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheIoManager.java
index 99878ec..9cb5c35 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheIoManager.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheIoManager.java
@@ -380,6 +380,9 @@ public class GridCacheIoManager extends GridCacheSharedManagerAdapter {
 
             unmarshall(nodeId, cacheMsg);
 
+//            if (!cacheMsg.partitionExchangeMessage())
+//                log.info("Message [from=" + nodeId + ", msg=" + cacheMsg + ']');
+
             if (cacheMsg.classError() != null)
                 processFailedMessage(nodeId, cacheMsg, c);
             else

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheSharedContext.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheSharedContext.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheSharedContext.java
index 989a810..39a3baa 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheSharedContext.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheSharedContext.java
@@ -732,7 +732,7 @@ public class GridCacheSharedContext<K, V> {
      * @param tx Transaction to close.
      * @throws IgniteCheckedException If failed.
      */
-    public void endTx(IgniteInternalTx tx) throws IgniteCheckedException {
+    public void endTx(GridNearTxLocal tx) throws IgniteCheckedException {
         tx.txState().awaitLastFut(this);
 
         tx.close();
@@ -749,7 +749,7 @@ public class GridCacheSharedContext<K, V> {
         if (ctx == null) {
             tx.txState().awaitLastFut(this);
 
-            return tx.commitTopLevelTxAsync();
+            return tx.commitNearTxLocalAsync();
         }
         else
             return ctx.cache().commitTxAsync(tx);
@@ -760,10 +760,10 @@ public class GridCacheSharedContext<K, V> {
      * @throws IgniteCheckedException If failed.
      * @return Rollback future.
      */
-    public IgniteInternalFuture rollbackTxAsync(IgniteInternalTx tx) throws IgniteCheckedException {
+    public IgniteInternalFuture rollbackTxAsync(GridNearTxLocal tx) throws IgniteCheckedException {
         tx.txState().awaitLastFut(this);
 
-        return tx.rollbackAsync();
+        return tx.rollbackNearTxLocalAsync();
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
index 77a99fc..7131612 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/GridCacheUtils.java
@@ -98,7 +98,6 @@ import org.jsr166.ConcurrentHashMap8;
 import static org.apache.ignite.IgniteSystemProperties.IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK;
 import static org.apache.ignite.cache.CacheAtomicityMode.TRANSACTIONAL;
 import static org.apache.ignite.cache.CacheMode.LOCAL;
-import static org.apache.ignite.cache.CacheMode.PARTITIONED;
 import static org.apache.ignite.cache.CacheMode.REPLICATED;
 import static org.apache.ignite.cache.CacheRebalanceMode.SYNC;
 import static org.apache.ignite.cache.CacheWriteSynchronizationMode.FULL_SYNC;
@@ -1261,7 +1260,7 @@ public class GridCacheUtils {
         try (GridNearTxLocal tx = cache.txStartEx(concurrency, isolation);) {
             clo.applyx(cache);
 
-            tx.commitTopLevelTx();
+            tx.commit();
         }
     }
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridCacheCommittedTxInfo.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridCacheCommittedTxInfo.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridCacheCommittedTxInfo.java
deleted file mode 100644
index 875ada0..0000000
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridCacheCommittedTxInfo.java
+++ /dev/null
@@ -1,117 +0,0 @@
-/*
- * 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.processors.cache.distributed;
-
-import java.io.Externalizable;
-import java.io.IOException;
-import java.io.ObjectInput;
-import java.io.ObjectOutput;
-import java.util.Collection;
-import java.util.Collections;
-import java.util.UUID;
-import org.apache.ignite.internal.processors.cache.transactions.IgniteInternalTx;
-import org.apache.ignite.internal.processors.cache.transactions.IgniteTxEntry;
-import org.apache.ignite.internal.processors.cache.version.GridCacheVersion;
-import org.apache.ignite.internal.util.tostring.GridToStringInclude;
-import org.apache.ignite.internal.util.typedef.internal.S;
-import org.apache.ignite.internal.util.typedef.internal.U;
-
-/**
- * Committed transaction information. Contains recovery writes that will be used to set commit values
- * in case if originating node crashes.
- */
-@Deprecated
-public class GridCacheCommittedTxInfo implements Externalizable {
-    /** */
-    private static final long serialVersionUID = 0L;
-
-    /** Originating transaction ID. */
-    private GridCacheVersion originatingTxId;
-
-    /** Originating node ID. */
-    private UUID originatingNodeId;
-
-    /** Recovery writes, i.e. values that have never been sent to remote nodes. */
-    @GridToStringInclude
-    private Collection<IgniteTxEntry> recoveryWrites;
-
-    /**
-     * Empty constructor required by {@link Externalizable}.
-     */
-    public GridCacheCommittedTxInfo() {
-        // No-op.
-    }
-
-    /**
-     * @param tx Committed cache transaction.
-     */
-    public GridCacheCommittedTxInfo(IgniteInternalTx tx) {
-        assert !tx.local() || !tx.replicated();
-
-        originatingTxId = tx.nearXidVersion();
-        originatingNodeId = tx.eventNodeId();
-    }
-
-    /**
-     * @return Originating transaction ID (the transaction ID for replicated cache and near transaction ID
-     *      for partitioned cache).
-     */
-    public GridCacheVersion originatingTxId() {
-        return originatingTxId;
-    }
-
-    /**
-     * @return Originating node ID (the local transaction node ID for replicated cache and near node ID
-     *      for partitioned cache).
-     */
-    public UUID originatingNodeId() {
-        return originatingNodeId;
-    }
-
-    /**
-     * @return Collection of recovery writes.
-     */
-    public Collection<IgniteTxEntry> recoveryWrites() {
-        return recoveryWrites == null ? Collections.<IgniteTxEntry>emptyList() : recoveryWrites;
-    }
-
-    /** {@inheritDoc} */
-    @Override public void writeExternal(ObjectOutput out) throws IOException {
-        originatingTxId.writeExternal(out);
-
-        U.writeUuid(out, originatingNodeId);
-
-        U.writeCollection(out, recoveryWrites);
-    }
-
-    /** {@inheritDoc} */
-    @Override public void readExternal(ObjectInput in) throws IOException, ClassNotFoundException {
-        originatingTxId = new GridCacheVersion();
-
-        originatingTxId.readExternal(in);
-
-        originatingNodeId = U.readUuid(in);
-
-        recoveryWrites = U.readCollection(in);
-    }
-
-    /** {@inheritDoc} */
-    @Override public String toString() {
-        return S.toString(GridCacheCommittedTxInfo.class, this, "recoveryWrites", recoveryWrites);
-    }
-}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxRemoteAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxRemoteAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxRemoteAdapter.java
index 5be1fe9..3fe59c0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxRemoteAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxRemoteAdapter.java
@@ -342,12 +342,6 @@ public abstract class GridDistributedTxRemoteAdapter extends IgniteTxAdapter
     }
 
     /** {@inheritDoc} */
-    @Override public IgniteInternalFuture<IgniteInternalTx> prepareAsync() {
-        assert false;
-        return null;
-    }
-
-    /** {@inheritDoc} */
     @Override public Set<IgniteTxKey> readSet() {
         return txState.readSet();
     }
@@ -401,15 +395,6 @@ public abstract class GridDistributedTxRemoteAdapter extends IgniteTxAdapter
     }
 
     /**
-     * Prepare phase.
-     *
-     * @throws IgniteCheckedException If prepare failed.
-     */
-    @Override public void prepare() throws IgniteCheckedException {
-        prepareRemoteTx();
-    }
-
-    /**
      * @throws IgniteCheckedException If commit failed.
      */
     @SuppressWarnings({"CatchGenericClass"})
@@ -756,11 +741,6 @@ public abstract class GridDistributedTxRemoteAdapter extends IgniteTxAdapter
         commitIfLocked();
     }
 
-    /** {@inheritDoc} */
-    @Override public void commit() throws IgniteCheckedException {
-        commitRemoteTx();
-    }
-
     /**
      * Forces commit for this tx.
      *
@@ -783,6 +763,35 @@ public abstract class GridDistributedTxRemoteAdapter extends IgniteTxAdapter
     }
 
     /** {@inheritDoc} */
+    @Override public IgniteInternalFuture<?>  salvageTx() {
+        try {
+            systemInvalidate(true);
+
+            prepareRemoteTx();
+
+            if (state() == PREPARING) {
+                if (log.isDebugEnabled())
+                    log.debug("Ignoring transaction in PREPARING state as it is currently handled " +
+                        "by another thread: " + this);
+
+                return null;
+            }
+
+            doneRemote(xidVersion(),
+                Collections.<GridCacheVersion>emptyList(),
+                Collections.<GridCacheVersion>emptyList(),
+                Collections.<GridCacheVersion>emptyList());
+
+            commitRemoteTx();
+        }
+        catch (IgniteCheckedException e) {
+            U.error(log, "Failed to invalidate transaction: " + xidVersion(), e);
+        }
+
+        return null;
+    }
+
+    /** {@inheritDoc} */
     @Override public void rollbackRemoteTx() {
         try {
             // Note that we don't evict near entries here -
@@ -801,12 +810,6 @@ public abstract class GridDistributedTxRemoteAdapter extends IgniteTxAdapter
     }
 
     /** {@inheritDoc} */
-    @SuppressWarnings({"CatchGenericClass"})
-    @Override public void rollback() {
-        rollbackRemoteTx();
-    }
-
-    /** {@inheritDoc} */
     @Override public IgniteInternalFuture<IgniteInternalTx> rollbackAsync() {
         rollbackRemoteTx();
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
index 0c63e45..1e09eda 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTransactionalCacheAdapter.java
@@ -178,7 +178,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
      * @throws GridDistributedLockCancelledException If lock has been cancelled.
      */
     @SuppressWarnings({"RedundantTypeArguments"})
-    @Nullable GridDhtTxRemote startRemoteTx(UUID nodeId,
+    @Nullable private GridDhtTxRemote startRemoteTx(UUID nodeId,
         GridDhtLockRequest req,
         GridDhtLockResponse res)
         throws IgniteCheckedException, GridDistributedLockCancelledException {
@@ -561,10 +561,10 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
 
         if (fail) {
             if (dhtTx != null)
-                dhtTx.rollback();
+                dhtTx.rollbackRemoteTx();
 
             if (nearTx != null) // Even though this should never happen, we leave this check for consistency.
-                nearTx.rollback();
+                nearTx.rollbackRemoteTx();
 
             List<KeyCacheObject> keys = req.keys();
 
@@ -961,8 +961,8 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
                             req.futureId(),
                             req.miniId(),
                             req.threadId(),
-                            req.implicitTx(),
-                            req.implicitSingleTx(),
+                            /*implicitTx*/false,
+                            /*implicitSingleTx*/false,
                             ctx.systemTx(),
                             false,
                             ctx.ioPolicy(),
@@ -989,7 +989,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
                             U.warn(log, msg);
 
                             if (tx != null)
-                                tx.rollback();
+                                tx.rollbackDhtLocal();
 
                             return new GridDhtFinishedFuture<>(new IgniteCheckedException(msg));
                         }
@@ -1086,7 +1086,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
 
             if (tx != null) {
                 try {
-                    tx.rollback();
+                    tx.rollbackDhtLocal();
                 }
                 catch (IgniteCheckedException ex) {
                     U.error(log, "Failed to rollback the transaction: " + tx, ex);
@@ -1290,7 +1290,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
      */
     private void sendLockReply(
         ClusterNode nearNode,
-        @Nullable IgniteInternalTx tx,
+        @Nullable GridDhtTxLocal tx,
         GridNearLockRequest req,
         GridNearLockResponse res
     ) {
@@ -1328,7 +1328,7 @@ public abstract class GridDhtTransactionalCacheAdapter<K, V> extends GridDhtCach
                 ", res=" + res + ']', e);
 
             if (tx != null)
-                tx.rollbackAsync();
+                tx.rollbackDhtLocalAsync();
 
             // Convert to closure exception as this method is only called form closures.
             throw new GridClosureException(e);

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocal.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocal.java
index 6dd40b0..b1c7e5b 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocal.java
@@ -292,82 +292,22 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
     }
 
     /** {@inheritDoc} */
-    @Override public IgniteInternalFuture<?> prepareAsync() {
-        if (optimistic()) {
-            assert isSystemInvalidate();
-
-            return prepareAsync(
-                null,
-                null,
-                Collections.<IgniteTxKey, GridCacheVersion>emptyMap(),
-                0,
-                nearMiniId,
-                null,
-                true);
-        }
-
-        long timeout = remainingTime();
-
-        // For pessimistic mode we don't distribute prepare request.
-        GridDhtTxPrepareFuture fut = prepFut;
-
-        if (fut == null) {
-            // Future must be created before any exception can be thrown.
-            if (!PREP_FUT_UPD.compareAndSet(this, null, fut = new GridDhtTxPrepareFuture(
-                cctx,
-                this,
-                timeout,
-                nearMiniId,
-                Collections.<IgniteTxKey, GridCacheVersion>emptyMap(),
-                true,
-                needReturnValue()))) {
-                if (timeout == -1)
-                    prepFut.onError(timeoutException());
+    @Override public IgniteInternalFuture<?> salvageTx() {
+        systemInvalidate(true);
 
-                return prepFut;
-            }
-        }
-        else
-            // Prepare was called explicitly.
-            return fut;
+        state(PREPARED);
 
-        if (!state(PREPARING)) {
-            if (setRollbackOnly()) {
-                if (timeout == -1)
-                    fut.onError(new IgniteTxTimeoutCheckedException("Transaction timed out and was rolled back: " +
-                        this));
-                else
-                    fut.onError(new IgniteCheckedException("Invalid transaction state for prepare [state=" + state() +
-                        ", tx=" + this + ']'));
-            }
-            else
-                fut.onError(new IgniteTxRollbackCheckedException("Invalid transaction state for prepare [state=" +
-                    state() + ", tx=" + this + ']'));
+        if (state() == PREPARING) {
+            if (log.isDebugEnabled())
+                log.debug("Ignoring transaction in PREPARING state as it is currently handled " +
+                    "by another thread: " + this);
 
-            return fut;
+            return null;
         }
 
-        try {
-            userPrepare();
-
-            if (!state(PREPARED)) {
-                setRollbackOnly();
-
-                fut.onError(new IgniteCheckedException("Invalid transaction state for commit [state=" + state() +
-                    ", tx=" + this + ']'));
-
-                return fut;
-            }
-
-            fut.complete();
-
-            return fut;
-        }
-        catch (IgniteCheckedException e) {
-            fut.onError(e);
+        setRollbackOnly();
 
-            return fut;
-        }
+        return rollbackDhtLocalAsync();
     }
 
     /**
@@ -382,7 +322,7 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
      * @param last {@code True} if this is last prepare request.
      * @return Future that will be completed when locks are acquired.
      */
-    public IgniteInternalFuture<GridNearTxPrepareResponse> prepareAsync(
+    public final IgniteInternalFuture<GridNearTxPrepareResponse> prepareAsync(
         @Nullable Collection<IgniteTxEntry> reads,
         @Nullable Collection<IgniteTxEntry> writes,
         Map<IgniteTxKey, GridCacheVersion> verMap,
@@ -478,7 +418,7 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
             fut.onError(new IgniteTxRollbackCheckedException("Failed to prepare transaction: " + this, e));
 
             try {
-                rollback();
+                rollbackDhtLocal();
             }
             catch (IgniteTxOptimisticCheckedException e1) {
                 if (log.isDebugEnabled())
@@ -523,7 +463,7 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
             if (prepFut != null)
                 prepFut.get(); // Check for errors.
 
-            boolean finished = finish(commit);
+            boolean finished = localFinish(commit);
 
             if (!finished)
                 err = new IgniteCheckedException("Failed to finish transaction [commit=" + commit +
@@ -544,9 +484,10 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
             fut.finish(commit);
     }
 
-    /** {@inheritDoc} */
-    @SuppressWarnings("unchecked")
-    @Override public IgniteInternalFuture<IgniteInternalTx> commitAsync() {
+    /**
+     * @return Commit future.
+     */
+    public IgniteInternalFuture<IgniteInternalTx> commitDhtLocalAsync() {
         if (log.isDebugEnabled())
             log.debug("Committing dht local tx: " + this);
 
@@ -577,15 +518,29 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
     }
 
     /** {@inheritDoc} */
+    @SuppressWarnings("unchecked")
+    @Override public IgniteInternalFuture<IgniteInternalTx> commitAsync() {
+        return commitDhtLocalAsync();
+    }
+
+    /** {@inheritDoc} */
     @Override protected void clearPrepareFuture(GridDhtTxPrepareFuture fut) {
         assert optimistic();
 
         PREP_FUT_UPD.compareAndSet(this, fut, null);
     }
 
-    /** {@inheritDoc} */
-    @SuppressWarnings("unchecked")
-    @Override public IgniteInternalFuture<IgniteInternalTx> rollbackAsync() {
+    /**
+     * @throws IgniteCheckedException If failed.
+     */
+    public void rollbackDhtLocal() throws IgniteCheckedException {
+        rollbackDhtLocalAsync().get();
+    }
+
+    /**
+     * @return Rollback future.
+     */
+    public IgniteInternalFuture<IgniteInternalTx> rollbackDhtLocalAsync() {
         final GridDhtTxFinishFuture fut = new GridDhtTxFinishFuture<>(cctx, this, false);
 
         cctx.mvcc().addFuture(fut, fut.futureId());
@@ -608,8 +563,13 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
     }
 
     /** {@inheritDoc} */
+    @Override public IgniteInternalFuture<IgniteInternalTx> rollbackAsync() {
+        return rollbackDhtLocalAsync();
+    }
+
+    /** {@inheritDoc} */
     @SuppressWarnings({"CatchGenericClass", "ThrowableInstanceNeverThrown"})
-    @Override public boolean finish(boolean commit) throws IgniteCheckedException {
+    @Override public boolean localFinish(boolean commit) throws IgniteCheckedException {
         assert nearFinFutId != null || isInvalidate() || !commit || isSystemInvalidate()
             || onePhaseCommit() || state() == PREPARED :
             "Invalid state [nearFinFutId=" + nearFinFutId + ", isInvalidate=" + isInvalidate() + ", commit=" + commit +
@@ -617,7 +577,7 @@ public class GridDhtTxLocal extends GridDhtTxLocalAdapter implements GridCacheMa
 
         assert nearMiniId != 0;
 
-        return super.finish(commit);
+        return super.localFinish(commit);
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocalAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocalAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocalAdapter.java
index 8e82c53..0329386 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocalAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxLocalAdapter.java
@@ -736,7 +736,7 @@ public abstract class GridDhtTxLocalAdapter extends IgniteTxLocalAdapter {
 
     /** {@inheritDoc} */
     @SuppressWarnings({"CatchGenericClass", "ThrowableInstanceNeverThrown"})
-    @Override public boolean finish(boolean commit) throws IgniteCheckedException {
+    @Override public boolean localFinish(boolean commit) throws IgniteCheckedException {
         if (log.isDebugEnabled())
             log.debug("Finishing dht local tx [tx=" + this + ", commit=" + commit + "]");
 
@@ -854,16 +854,6 @@ public abstract class GridDhtTxLocalAdapter extends IgniteTxLocalAdapter {
     }
 
     /** {@inheritDoc} */
-    @Override public void rollback() throws IgniteCheckedException {
-        try {
-            rollbackAsync().get();
-        }
-        finally {
-            cctx.tm().resetContext();
-        }
-    }
-
-    /** {@inheritDoc} */
     @Override public String toString() {
         return GridToStringBuilder.toString(GridDhtTxLocalAdapter.class, this, "nearNodes", nearMap.keySet(),
             "dhtNodes", dhtMap.keySet(), "explicitLock", explicitLock, "super", super.toString());

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxPrepareFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxPrepareFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxPrepareFuture.java
index 56884ff..ae570aa 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxPrepareFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/GridDhtTxPrepareFuture.java
@@ -718,18 +718,8 @@ public final class GridDhtTxPrepareFuture extends GridCompoundFuture<IgniteInter
                     CIX1<IgniteInternalFuture<IgniteInternalTx>> resClo =
                         new CIX1<IgniteInternalFuture<IgniteInternalTx>>() {
                             @Override public void applyx(IgniteInternalFuture<IgniteInternalTx> fut) {
-                                try {
-                                    if (REPLIED_UPD.compareAndSet(GridDhtTxPrepareFuture.this, 0, 1))
-                                        sendPrepareResponse(res);
-                                }
-                                catch (IgniteCheckedException e) {
-                                    U.error(log, "Failed to send prepare response [txId=" + tx.nearXidVersion() + "," +
-                                        ", dhtTxId=" + tx.xidVersion() +
-                                        ", node=" + tx.nearNodeId() +
-                                        ", res=" + res,
-                                        ", tx=" + tx,
-                                        e);
-                                }
+                                if (REPLIED_UPD.compareAndSet(GridDhtTxPrepareFuture.this, 0, 1))
+                                    sendPrepareResponse(res);
                             }
                         };
 
@@ -761,18 +751,8 @@ public final class GridDhtTxPrepareFuture extends GridCompoundFuture<IgniteInter
                 }
             }
             else {
-                try {
-                    if (REPLIED_UPD.compareAndSet(this, 0, 1))
-                        sendPrepareResponse(res);
-                }
-                catch (IgniteCheckedException e) {
-                    U.error(log, "Failed to send prepare response [txId=" + tx.nearXidVersion() + "," +
-                        ", dhtTxId=" + tx.xidVersion() +
-                        ", node=" + tx.nearNodeId() +
-                        ", res=" + res,
-                        ", tx=" + tx,
-                        e);
-                }
+                if (REPLIED_UPD.compareAndSet(this, 0, 1))
+                    sendPrepareResponse(res);
             }
 
             return true;
@@ -784,14 +764,6 @@ public final class GridDhtTxPrepareFuture extends GridCompoundFuture<IgniteInter
                 try {
                     sendPrepareResponse(res);
                 }
-                catch (IgniteCheckedException e) {
-                    U.error(log, "Failed to send prepare response [txId=" + tx.nearXidVersion() + "," +
-                        ", dhtTxId=" + tx.xidVersion() +
-                        ", node=" + tx.nearNodeId() +
-                        ", res=" + res,
-                        ", tx=" + tx,
-                        e);
-                }
                 finally {
                     // Will call super.onDone().
                     onComplete(res);
@@ -819,9 +791,8 @@ public final class GridDhtTxPrepareFuture extends GridCompoundFuture<IgniteInter
 
     /**
      * @param res Response.
-     * @throws IgniteCheckedException If failed to send response.
      */
-    private void sendPrepareResponse(GridNearTxPrepareResponse res) throws IgniteCheckedException {
+    private void sendPrepareResponse(GridNearTxPrepareResponse res) {
         if (!tx.nearNodeId().equals(cctx.localNodeId())) {
             Throwable err = this.err;
 
@@ -837,13 +808,31 @@ public final class GridDhtTxPrepareFuture extends GridCompoundFuture<IgniteInter
                 return;
             }
 
-            cctx.io().send(tx.nearNodeId(), res, tx.ioPolicy());
+            try {
+                cctx.io().send(tx.nearNodeId(), res, tx.ioPolicy());
 
-            if (msgLog.isDebugEnabled()) {
-                msgLog.debug("DHT prepare fut, sent response [txId=" + tx.nearXidVersion() +
-                    ", dhtTxId=" + tx.xidVersion() +
-                    ", node=" + tx.nearNodeId() +
-                    ", res=" + res + ']');
+                if (msgLog.isDebugEnabled()) {
+                    msgLog.debug("DHT prepare fut, sent response [txId=" + tx.nearXidVersion() +
+                        ", dhtTxId=" + tx.xidVersion() +
+                        ", node=" + tx.nearNodeId() +
+                        ", res=" + res + ']');
+                }
+            }
+            catch (ClusterTopologyCheckedException e) {
+                if (msgLog.isDebugEnabled()) {
+                    msgLog.debug("Failed to send prepare response, node left [txId=" + tx.nearXidVersion() + "," +
+                        ", dhtTxId=" + tx.xidVersion() +
+                        ", node=" + tx.nearNodeId() +
+                        ", res=" + res + ']');
+                }
+            }
+            catch (IgniteCheckedException e) {
+                U.error(msgLog, "Failed to send prepare response [txId=" + tx.nearXidVersion() + "," +
+                        ", dhtTxId=" + tx.xidVersion() +
+                        ", node=" + tx.nearNodeId() +
+                        ", res=" + res,
+                    ", tx=" + tx + ']',
+                    e);
             }
         }
     }

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/colocated/GridDhtColocatedLockFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/colocated/GridDhtColocatedLockFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/colocated/GridDhtColocatedLockFuture.java
index 0ce380d..79c15fb 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/colocated/GridDhtColocatedLockFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/colocated/GridDhtColocatedLockFuture.java
@@ -917,6 +917,8 @@ public final class GridDhtColocatedLockFuture extends GridCompoundIdentityFuture
                                         first = false;
                                     }
 
+                                    assert !implicitTx() && !implicitSingleTx() : tx;
+
                                     req = new GridNearLockRequest(
                                         cctx.cacheId(),
                                         topVer,
@@ -925,8 +927,6 @@ public final class GridDhtColocatedLockFuture extends GridCompoundIdentityFuture
                                         futId,
                                         lockVer,
                                         inTx(),
-                                        implicitTx(),
-                                        implicitSingleTx(),
                                         read,
                                         retval,
                                         isolation(),
@@ -982,9 +982,6 @@ public final class GridDhtColocatedLockFuture extends GridCompoundIdentityFuture
                 }
             }
 
-            if (inTx() && req != null)
-                req.hasTransforms(tx.hasTransforms());
-
             if (!distributedKeys.isEmpty()) {
                 mapping.distributedKeys(distributedKeys);
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockFuture.java
index ffc84d8..1948df0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockFuture.java
@@ -1045,6 +1045,8 @@ public final class GridNearLockFuture extends GridCompoundIdentityFuture<Boolean
                                                 first = false;
                                             }
 
+                                            assert !implicitTx() && !implicitSingleTx() : tx;
+
                                             req = new GridNearLockRequest(
                                                 cctx.cacheId(),
                                                 topVer,
@@ -1053,8 +1055,6 @@ public final class GridNearLockFuture extends GridCompoundIdentityFuture<Boolean
                                                 futId,
                                                 lockVer,
                                                 inTx(),
-                                                implicitTx(),
-                                                implicitSingleTx(),
                                                 read,
                                                 retval,
                                                 isolation(),

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockRequest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockRequest.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockRequest.java
index 229961e..48b508b 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockRequest.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearLockRequest.java
@@ -46,6 +46,15 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
     /** */
     private static final long serialVersionUID = 0L;
 
+    /** */
+    private static final int NEED_RETURN_VALUE_FLAG_MASK = 0x01;
+
+    /** */
+    private static final int FIRST_CLIENT_REQ_FLAG_MASK = 0x02;
+
+    /** */
+    private static final int SYNC_COMMIT_FLAG_MASK = 0x04;
+
     /** Topology version. */
     private AffinityTopologyVersion topVer;
 
@@ -55,15 +64,6 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
     /** Filter. */
     private CacheEntryPredicate[] filter;
 
-    /** Implicit flag. */
-    private boolean implicitTx;
-
-    /** Implicit transaction with one key flag. */
-    private boolean implicitSingleTx;
-
-    /** Flag is kept for backward compatibility. */
-    private boolean onePhaseCommit;
-
     /** Array of mapped DHT versions for this entry. */
     @GridToStringInclude
     private GridCacheVersion[] dhtVers;
@@ -74,23 +74,14 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
     /** Task name hash. */
     private int taskNameHash;
 
-    /** Has transforms flag. */
-    private boolean hasTransforms;
-
-    /** Sync commit flag. */
-    private boolean syncCommit;
-
     /** TTL for create operation. */
     private long createTtl;
 
     /** TTL for read operation. */
     private long accessTtl;
 
-    /** Flag indicating whether cache operation requires a previous value. */
-    private boolean retVal;
-
-    /** {@code True} if first lock request for lock operation sent from client node. */
-    private boolean firstClientReq;
+    /** */
+    private byte flags;
 
     /**
      * Empty constructor required for {@link Externalizable}.
@@ -107,8 +98,6 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
      * @param futId Future ID.
      * @param lockVer Cache version.
      * @param isInTx {@code True} if implicit transaction lock.
-     * @param implicitTx Flag to indicate that transaction is implicit.
-     * @param implicitSingleTx Implicit-transaction-with-one-key flag.
      * @param isRead Indicates whether implicit lock is for read or write operation.
      * @param retVal Return value flag.
      * @param isolation Transaction isolation.
@@ -133,8 +122,6 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
         IgniteUuid futId,
         GridCacheVersion lockVer,
         boolean isInTx,
-        boolean implicitTx,
-        boolean implicitSingleTx,
         boolean isRead,
         boolean retVal,
         TransactionIsolation isolation,
@@ -174,24 +161,43 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
         assert topVer.compareTo(AffinityTopologyVersion.ZERO) > 0;
 
         this.topVer = topVer;
-        this.implicitTx = implicitTx;
-        this.implicitSingleTx = implicitSingleTx;
-        this.syncCommit = syncCommit;
         this.subjId = subjId;
         this.taskNameHash = taskNameHash;
         this.createTtl = createTtl;
         this.accessTtl = accessTtl;
-        this.retVal = retVal;
-        this.firstClientReq = firstClientReq;
 
         dhtVers = new GridCacheVersion[keyCnt];
+
+        setFlag(syncCommit, SYNC_COMMIT_FLAG_MASK);
+        setFlag(firstClientReq, FIRST_CLIENT_REQ_FLAG_MASK);
+        setFlag(retVal, NEED_RETURN_VALUE_FLAG_MASK);
+    }
+
+    /**
+     * Sets flag mask.
+     *
+     * @param flag Set or clear.
+     * @param mask Mask.
+     */
+    private void setFlag(boolean flag, int mask) {
+        flags = flag ? (byte)(flags | mask) : (byte)(flags & ~mask);
+    }
+
+    /**
+     * Reags flag mask.
+     *
+     * @param mask Mask to read.
+     * @return Flag value.
+     */
+    private boolean isFlag(int mask) {
+        return (flags & mask) != 0;
     }
 
     /**
      * @return {@code True} if first lock request for lock operation sent from client node.
      */
     public boolean firstClientRequest() {
-        return firstClientReq;
+        return isFlag(FIRST_CLIENT_REQ_FLAG_MASK);
     }
 
     /**
@@ -216,24 +222,10 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
     }
 
     /**
-     * @return Implicit transaction flag.
-     */
-    public boolean implicitTx() {
-        return implicitTx;
-    }
-
-    /**
-     * @return Implicit-transaction-with-one-key flag.
-     */
-    public boolean implicitSingleTx() {
-        return implicitSingleTx;
-    }
-
-    /**
      * @return Sync commit flag.
      */
     public boolean syncCommit() {
-        return syncCommit;
+        return isFlag(SYNC_COMMIT_FLAG_MASK);
     }
 
     /**
@@ -268,24 +260,10 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
     }
 
     /**
-     * @param hasTransforms {@code True} if originating transaction has transform entries.
-     */
-    public void hasTransforms(boolean hasTransforms) {
-        this.hasTransforms = hasTransforms;
-    }
-
-    /**
-     * @return {@code True} if originating transaction has transform entries.
-     */
-    public boolean hasTransforms() {
-        return hasTransforms;
-    }
-
-    /**
      * @return Need return value flag.
      */
     public boolean needReturnValue() {
-        return retVal;
+        return isFlag(NEED_RETURN_VALUE_FLAG_MASK);
     }
 
     /**
@@ -399,66 +377,30 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
                 writer.incrementState();
 
             case 24:
-                if (!writer.writeBoolean("firstClientReq", firstClientReq))
+                if (!writer.writeByte("flags", flags))
                     return false;
 
                 writer.incrementState();
 
             case 25:
-                if (!writer.writeBoolean("hasTransforms", hasTransforms))
-                    return false;
-
-                writer.incrementState();
-
-            case 26:
-                if (!writer.writeBoolean("implicitSingleTx", implicitSingleTx))
-                    return false;
-
-                writer.incrementState();
-
-            case 27:
-                if (!writer.writeBoolean("implicitTx", implicitTx))
-                    return false;
-
-                writer.incrementState();
-
-            case 28:
                 if (!writer.writeInt("miniId", miniId))
                     return false;
 
                 writer.incrementState();
 
-            case 29:
-                if (!writer.writeBoolean("onePhaseCommit", onePhaseCommit))
-                    return false;
-
-                writer.incrementState();
-
-            case 30:
-                if (!writer.writeBoolean("retVal", retVal))
-                    return false;
-
-                writer.incrementState();
-
-            case 31:
+            case 26:
                 if (!writer.writeUuid("subjId", subjId))
                     return false;
 
                 writer.incrementState();
 
-            case 32:
-                if (!writer.writeBoolean("syncCommit", syncCommit))
-                    return false;
-
-                writer.incrementState();
-
-            case 33:
+            case 27:
                 if (!writer.writeInt("taskNameHash", taskNameHash))
                     return false;
 
                 writer.incrementState();
 
-            case 34:
+            case 28:
                 if (!writer.writeMessage("topVer", topVer))
                     return false;
 
@@ -513,7 +455,7 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
                 reader.incrementState();
 
             case 24:
-                firstClientReq = reader.readBoolean("firstClientReq");
+                flags = reader.readByte("flags");
 
                 if (!reader.isLastRead())
                     return false;
@@ -521,30 +463,6 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
                 reader.incrementState();
 
             case 25:
-                hasTransforms = reader.readBoolean("hasTransforms");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 26:
-                implicitSingleTx = reader.readBoolean("implicitSingleTx");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 27:
-                implicitTx = reader.readBoolean("implicitTx");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 28:
                 miniId = reader.readInt("miniId");
 
                 if (!reader.isLastRead())
@@ -552,23 +470,7 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
 
                 reader.incrementState();
 
-            case 29:
-                onePhaseCommit = reader.readBoolean("onePhaseCommit");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 30:
-                retVal = reader.readBoolean("retVal");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 31:
+            case 26:
                 subjId = reader.readUuid("subjId");
 
                 if (!reader.isLastRead())
@@ -576,15 +478,7 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
 
                 reader.incrementState();
 
-            case 32:
-                syncCommit = reader.readBoolean("syncCommit");
-
-                if (!reader.isLastRead())
-                    return false;
-
-                reader.incrementState();
-
-            case 33:
+            case 27:
                 taskNameHash = reader.readInt("taskNameHash");
 
                 if (!reader.isLastRead())
@@ -592,7 +486,7 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
 
                 reader.incrementState();
 
-            case 34:
+            case 28:
                 topVer = reader.readMessage("topVer");
 
                 if (!reader.isLastRead())
@@ -612,7 +506,7 @@ public class GridNearLockRequest extends GridDistributedLockRequest {
 
     /** {@inheritDoc} */
     @Override public byte fieldsCount() {
-        return 35;
+        return 29;
     }
 
     /** {@inheritDoc} */

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearOptimisticTxPrepareFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearOptimisticTxPrepareFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearOptimisticTxPrepareFuture.java
index cd3b0ea..6189b38 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearOptimisticTxPrepareFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearOptimisticTxPrepareFuture.java
@@ -160,7 +160,7 @@ public class GridNearOptimisticTxPrepareFuture extends GridNearOptimisticTxPrepa
             if (e instanceof IgniteTxRollbackCheckedException) {
                 if (marked) {
                     try {
-                        tx.rollbackTopLevelTx();
+                        tx.rollback();
                     }
                     catch (IgniteCheckedException ex) {
                         U.error(log, "Failed to automatically rollback transaction: " + tx, ex);

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxFinishFuture.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxFinishFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxFinishFuture.java
index 7387501..1b0566b 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxFinishFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxFinishFuture.java
@@ -310,7 +310,7 @@ public final class GridNearTxFinishFuture<K, V> extends GridCompoundIdentityFutu
                         err = new TransactionRollbackException("Failed to commit transaction.", err);
 
                     try {
-                        tx.finish(err == null);
+                        tx.localFinish(err == null);
                     }
                     catch (IgniteCheckedException e) {
                         if (err != null)
@@ -402,7 +402,7 @@ public final class GridNearTxFinishFuture<K, V> extends GridCompoundIdentityFutu
         }
 
         try {
-            if (tx.finish(commit) || (!commit && tx.state() == UNKNOWN)) {
+            if (tx.localFinish(commit) || (!commit && tx.state() == UNKNOWN)) {
                 if ((tx.onePhaseCommit() && needFinishOnePhase(commit)) || (!tx.onePhaseCommit() && mappings != null)) {
                     if (mappings.single()) {
                         GridDistributedTxMapping mapping = mappings.singleMapping();

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxLocal.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxLocal.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxLocal.java
index 5a5470b..81606d4 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxLocal.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxLocal.java
@@ -117,7 +117,7 @@ import static org.apache.ignite.transactions.TransactionState.UNKNOWN;
  * Replicated user transaction.
  */
 @SuppressWarnings("unchecked")
-public class GridNearTxLocal extends GridDhtTxLocalAdapter {
+public class GridNearTxLocal extends GridDhtTxLocalAdapter implements AutoCloseable  {
     /** */
     private static final long serialVersionUID = 0L;
 
@@ -169,6 +169,9 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
     /** */
     private boolean hasRemoteLocks;
 
+    /** If this transaction contains transform entries. */
+    protected boolean transform;
+
     /** */
     @GridToStringExclude
     private TransactionProxyImpl proxy;
@@ -687,7 +690,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
         if (F.isEmpty(map0) && F.isEmpty(invokeMap0)) {
             if (implicit())
                 try {
-                    commitTopLevelTx();
+                    commit();
                 }
                 catch (IgniteCheckedException e) {
                     return new GridFinishedFuture<>(e);
@@ -965,7 +968,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
             for (Object key : keys) {
                 if (key == null) {
-                    rollbackTopLevelTx();
+                    rollback();
 
                     throw new NullPointerException("Null key.");
                 }
@@ -1473,7 +1476,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
         if (F.isEmpty(keys0)) {
             if (implicit()) {
                 try {
-                    commitTopLevelTx();
+                    commit();
                 }
                 catch (IgniteCheckedException e) {
                     return new GridFinishedFuture<>(e);
@@ -1604,7 +1607,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
                 // with prepare response, if required.
                 assert loadFut.isDone();
 
-                return nonInterruptable(commitTopLevelTxAsync().chain(new CX1<IgniteInternalFuture<IgniteInternalTx>, GridCacheReturn>() {
+                return nonInterruptable(commitNearTxLocalAsync().chain(new CX1<IgniteInternalFuture<IgniteInternalTx>, GridCacheReturn>() {
                     @Override public GridCacheReturn applyx(IgniteInternalFuture<IgniteInternalTx> txFut)
                         throws IgniteCheckedException {
                         try {
@@ -1614,7 +1617,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
                                 implicitRes.value(), implicitRes.success());
                         }
                         catch (IgniteCheckedException | RuntimeException e) {
-                            rollbackAsync();
+                            rollbackNearTxLocalAsync();
 
                             throw e;
                         }
@@ -2371,7 +2374,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
                 return new GridFinishedFuture<>(e);
             }
 
-            return nonInterruptable(commitTopLevelTxAsync().chain(
+            return nonInterruptable(commitNearTxLocalAsync().chain(
                 new CX1<IgniteInternalFuture<IgniteInternalTx>, GridCacheReturn>() {
                     @Override public GridCacheReturn applyx(IgniteInternalFuture<IgniteInternalTx> txFut)
                         throws IgniteCheckedException {
@@ -2390,7 +2393,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
                         }
                         catch (IgniteCheckedException | RuntimeException e) {
                             if (!(e instanceof NodeStoppingException))
-                                rollbackAsync();
+                                rollbackNearTxLocalAsync();
 
                             throw e;
                         }
@@ -2986,7 +2989,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
     /** {@inheritDoc} */
     @SuppressWarnings({"CatchGenericClass", "ThrowableInstanceNeverThrown"})
-    @Override public boolean finish(boolean commit) throws IgniteCheckedException {
+    @Override public boolean localFinish(boolean commit) throws IgniteCheckedException {
         if (log.isDebugEnabled())
             log.debug("Finishing near local tx [tx=" + this + ", commit=" + commit + "]");
 
@@ -3070,7 +3073,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
     /**
      * @return Tx prepare future.
      */
-    public IgniteInternalFuture<?> prepareTopLevelTx() {
+    public IgniteInternalFuture<?> prepareNearTxLocal() {
         GridNearTxPrepareFutureAdapter fut = (GridNearTxPrepareFutureAdapter)prepFut;
 
         if (fut == null) {
@@ -3106,21 +3109,37 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
     }
 
     /** {@inheritDoc} */
-    @Override public IgniteInternalFuture<?> prepareAsync() {
-        return prepareTopLevelTx();
+    @Override public IgniteInternalFuture<?> salvageTx() {
+        assert false : "Should not be called for GridNearTxLocal";
+
+        return null;
+    }
+
+    /**
+     * @throws IgniteCheckedException If failed.
+     */
+    public final void prepare() throws IgniteCheckedException {
+        prepareAsync().get();
+    }
+
+    /**
+     * @return Prepare future.
+     */
+    private IgniteInternalFuture<?> prepareAsync() {
+        return prepareNearTxLocal();
     }
 
     /**
      * @throws IgniteCheckedException If failed.
      */
-    public void commitTopLevelTx() throws IgniteCheckedException {
-        commitTopLevelTxAsync().get();
+    public void commit() throws IgniteCheckedException {
+        commitNearTxLocalAsync().get();
     }
 
     /**
      * @return Finish future.
      */
-    public IgniteInternalFuture<IgniteInternalTx> commitTopLevelTxAsync() {
+    public IgniteInternalFuture<IgniteInternalTx> commitNearTxLocalAsync() {
         if (log.isDebugEnabled())
             log.debug("Committing near local tx: " + this);
 
@@ -3136,7 +3155,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
             return new GridFinishedFuture<>((IgniteInternalTx)this);
         }
 
-        final IgniteInternalFuture<?> prepareFut = prepareTopLevelTx();
+        final IgniteInternalFuture<?> prepareFut = prepareNearTxLocal();
 
         GridNearTxFinishFuture fut = commitFut;
 
@@ -3177,20 +3196,20 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
     /** {@inheritDoc} */
     @Override public IgniteInternalFuture<IgniteInternalTx> commitAsync() {
-        return commitTopLevelTxAsync();
+        return commitNearTxLocalAsync();
     }
 
     /**
      * @throws IgniteCheckedException If failed.
      */
-    public void rollbackTopLevelTx() throws IgniteCheckedException {
-        rollbackTopLevelTxAsync().get();
+    public void rollback() throws IgniteCheckedException {
+        rollbackNearTxLocalAsync().get();
     }
 
     /**
      * @return Rollback future.
      */
-    public IgniteInternalFuture<IgniteInternalTx> rollbackTopLevelTxAsync() {
+    public IgniteInternalFuture<IgniteInternalTx> rollbackNearTxLocalAsync() {
         if (log.isDebugEnabled())
             log.debug("Rolling back near tx: " + this);
 
@@ -3255,7 +3274,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
     /** {@inheritDoc} */
     @Override public IgniteInternalFuture<IgniteInternalTx> rollbackAsync() {
-        return rollbackTopLevelTxAsync();
+        return rollbackNearTxLocalAsync();
     }
 
     /**
@@ -3332,7 +3351,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
             fut.onError(new IgniteTxRollbackCheckedException("Failed to prepare transaction: " + this, e));
 
             try {
-                rollbackTopLevelTx();
+                rollback();
             }
             catch (IgniteTxOptimisticCheckedException e1) {
                 if (log.isDebugEnabled())
@@ -3625,7 +3644,24 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
     /** {@inheritDoc} */
     @Override public void close() throws IgniteCheckedException {
-        super.close();
+        TransactionState state = state();
+
+        if (state != ROLLING_BACK && state != ROLLED_BACK && state != COMMITTING && state != COMMITTED)
+            rollback();
+
+        synchronized (this) {
+            try {
+                while (!done())
+                    wait();
+            }
+            catch (InterruptedException e) {
+                Thread.currentThread().interrupt();
+
+                if (!done())
+                    throw new IgniteCheckedException("Got interrupted while waiting for transaction to complete: " +
+                        this, e);
+            }
+        }
 
         if (accessMap != null) {
             assert optimistic();
@@ -3651,8 +3687,10 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
         return prepFut;
     }
 
-    /** {@inheritDoc} */
-    @Override public void onRemap(AffinityTopologyVersion topVer) {
+    /**
+     * @param topVer New topology version.
+     */
+    public void onRemap(AffinityTopologyVersion topVer) {
         assert cctx.kernalContext().clientNode();
 
         mapped = false;
@@ -3933,7 +3971,7 @@ public class GridNearTxLocal extends GridDhtTxLocalAdapter {
 
                 // Commit implicit transactions.
                 if (implicit())
-                    commitTopLevelTx();
+                    commit();
 
                 rollback = false;
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareFutureAdapter.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareFutureAdapter.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareFutureAdapter.java
index a0f28c5..f9a6353 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareFutureAdapter.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareFutureAdapter.java
@@ -38,7 +38,6 @@ import org.apache.ignite.internal.processors.cache.version.GridCacheVersion;
 import org.apache.ignite.internal.util.future.GridCompoundFuture;
 import org.apache.ignite.internal.util.tostring.GridToStringExclude;
 import org.apache.ignite.internal.util.tostring.GridToStringInclude;
-import org.apache.ignite.internal.util.typedef.F;
 import org.apache.ignite.internal.util.typedef.internal.CU;
 import org.apache.ignite.internal.util.typedef.internal.U;
 import org.apache.ignite.lang.IgniteReducer;
@@ -160,8 +159,8 @@ public abstract class GridNearTxPrepareFutureAdapter extends
      *
      * @param txMapping Transaction mapping.
      */
-    protected final void checkOnePhase(GridDhtTxMapping txMapping) {
-        if (tx.storeUsed())
+    final void checkOnePhase(GridDhtTxMapping txMapping) {
+        if (tx.storeWriteThrough())
             return;
 
         Map<UUID, Collection<UUID>> map = txMapping.transactionNodes();

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareRequest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareRequest.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareRequest.java
index ffeeb51..5b0807f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareRequest.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxPrepareRequest.java
@@ -103,7 +103,7 @@ public class GridNearTxPrepareRequest extends GridDistributedTxPrepareRequest {
     public GridNearTxPrepareRequest(
         IgniteUuid futId,
         AffinityTopologyVersion topVer,
-        IgniteInternalTx tx,
+        GridNearTxLocal tx,
         long timeout,
         Collection<IgniteTxEntry> reads,
         Collection<IgniteTxEntry> writes,

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxRemote.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxRemote.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxRemote.java
index 4f4be57..c961f6f 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxRemote.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearTxRemote.java
@@ -115,7 +115,7 @@ public class GridNearTxRemote extends GridDistributedTxRemoteAdapter {
             ctx, 
             nodeId, 
             xidVer,
-            commitVer, 
+            commitVer,
             sys, 
             plc, 
             concurrency, 
@@ -289,7 +289,7 @@ public class GridNearTxRemote extends GridDistributedTxRemoteAdapter {
      *
      * @param key Evicted key.
      */
-    public void addEvicted(IgniteTxKey key) {
+    void addEvicted(IgniteTxKey key) {
         evicted.add(key);
     }
 

http://git-wip-us.apache.org/repos/asf/ignite/blob/2272fadc/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteInternalTx.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteInternalTx.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteInternalTx.java
index d26696e..6d1e0c0 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteInternalTx.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/IgniteInternalTx.java
@@ -21,7 +21,6 @@ import java.util.Collection;
 import java.util.Map;
 import java.util.Set;
 import java.util.UUID;
-import org.apache.ignite.IgniteCheckedException;
 import org.apache.ignite.cache.CacheWriteSynchronizationMode;
 import org.apache.ignite.internal.IgniteInternalFuture;
 import org.apache.ignite.internal.processors.affinity.AffinityTopologyVersion;
@@ -44,7 +43,7 @@ import org.jetbrains.annotations.Nullable;
 /**
  * Transaction managed by cache ({@code 'Ex'} stands for external).
  */
-public interface IgniteInternalTx extends AutoCloseable {
+public interface IgniteInternalTx {
     /**
      *
      */
@@ -179,27 +178,6 @@ public interface IgniteInternalTx extends AutoCloseable {
     public boolean isRollbackOnly();
 
     /**
-     * Commits this transaction by initiating {@code two-phase-commit} process.
-     *
-     * @throws IgniteCheckedException If commit failed.
-     */
-    public void commit() throws IgniteCheckedException;
-
-    /**
-     * Ends the transaction. Transaction will be rolled back if it has not been committed.
-     *
-     * @throws IgniteCheckedException If transaction could not be gracefully ended.
-     */
-    @Override public void close() throws IgniteCheckedException;
-
-    /**
-     * Rolls back this transaction.
-     *
-     * @throws IgniteCheckedException If rollback failed.
-     */
-    public void rollback() throws IgniteCheckedException;
-
-    /**
      * Removes metadata by key.
      *
      * @param key Key of the metadata to remove.
@@ -242,7 +220,7 @@ public interface IgniteInternalTx extends AutoCloseable {
      * @return {@code True} if transaction is allowed to use store and transactions spans one or more caches with
      *      store enabled.
      */
-    public boolean storeUsed();
+    public boolean storeWriteThrough();
 
     /**
      * Checks if this is system cache transaction. System transactions are isolated from user transactions
@@ -300,11 +278,6 @@ public interface IgniteInternalTx extends AutoCloseable {
     public boolean empty();
 
     /**
-     * @return {@code True} if preparing flag was set with this call.
-     */
-    public boolean markPreparing();
-
-    /**
      * @param status Finalization status to set.
      * @return {@code True} if could mark was set.
      */
@@ -399,11 +372,6 @@ public interface IgniteInternalTx extends AutoCloseable {
     public boolean local();
 
     /**
-     * @return {@code True} if transaction is replicated.
-     */
-    public boolean replicated();
-
-    /**
      * @return Subject ID initiated this transaction.
      */
     public UUID subjectId();
@@ -426,11 +394,6 @@ public interface IgniteInternalTx extends AutoCloseable {
     public boolean user();
 
     /**
-     * @return Transaction write synchronization mode.
-     */
-    public CacheWriteSynchronizationMode syncMode();
-
-    /**
      * @param key Key to check.
      * @return {@code True} if key is present.
      */
@@ -518,18 +481,9 @@ public interface IgniteInternalTx extends AutoCloseable {
     public void commitVersion(GridCacheVersion commitVer);
 
     /**
-     * Prepare state.
-     *
-     * @throws IgniteCheckedException If failed.
-     */
-    public void prepare() throws IgniteCheckedException;
-
-    /**
-     * Prepare stage.
-     *
-     * @return Future for prepare step.
+     * @return Future.
      */
-    public IgniteInternalFuture<?> prepareAsync();
+    @Nullable public IgniteInternalFuture<?> salvageTx();
 
     /**
      * @param endVer End version (a.k.a. <tt>'tnc'</tt> or <tt>'transaction number counter'</tt>)
@@ -663,7 +617,8 @@ public interface IgniteInternalTx extends AutoCloseable {
      * @param committed Committed transactions relative to base.
      * @param rolledback Rolled back transactions relative to base.
      */
-    public void completedVersions(GridCacheVersion base, Collection<GridCacheVersion> committed,
+    public void completedVersions(GridCacheVersion base,
+        Collection<GridCacheVersion> committed,
         Collection<GridCacheVersion> rolledback);
 
     /**
@@ -677,17 +632,6 @@ public interface IgniteInternalTx extends AutoCloseable {
     public boolean onePhaseCommit();
 
     /**
-     * @return {@code True} if transaction has transform entries. This flag will be only set for local
-     *      transactions.
-     */
-    public boolean hasTransforms();
-
-    /**
-     * @param topVer New topology version.
-     */
-    public void onRemap(AffinityTopologyVersion topVer);
-
-    /**
      * @param e Commit error.
      */
     public void commitError(Throwable e);