You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@ignite.apache.org by Semyon Boikov <sb...@gridgain.com> on 2015/10/15 09:00:24 UTC

Optimistic/serializable transactions implementation

Hello,

I'm working on new implementation for optimistic/serializable transaction
mode (current implementation is inefficient and have bugs). This mode is
supposed to be used when concurrent transactions do not update the same
keys, in this case transactions can be executed more efficiently, but if
concurrent transactions have read of write conflicts then
TransactionOptimisticException can be thrown and transaction should be
retried. Also with current transactions implementation user should order
updated keys, otherwise deadlock is possible, we want to remove this
requirement for optimistic/serializable mode.

Issue with read/write conflicts can be detected if when read is performed
entry version is stored and then compared with current version when
transaction lock is acquired.

Another issue is avoid deadlocks when transactions acquire keys in
different order.

I created one possible solution using 'try-lock' approach: for each cache
entry we already have queue with current lock owner and transactions trying
to acquire entry lock.
When optimistic/serializable transaction tries to acquire entry lock it
checks that entry is not locked and there are no others transactions
waiting for lock entry, otherwise transaction fails with
TransactionOptimisticException. But this approach does not work well when
two transaction have lot of intersecting keys: it is possible that these
transaction will constantly conflict and will both constantly fail with
TransactionOptimisticException.

It seems there is better approach to resolve these conflicts to do not fail
all conflicting transactions:
- we should order all transactions by some attribute (e.g. all transactions
already have unique version)
- transaction with greater order should always 'win' transaction with lower
order
- per-entry queue with waiting transactions should be sorted by this order
- when transaction tries to acquire entry lock it is added in waiting queue
if queue is empty or last waiting transaction have lower order, otherwise
transaction fails

With this approach transaction lock assignment is ordered and transactions
with lower order never wait for transactions with greater order, so this
algorithm should not cause deadlocks.

I also created unit test simulating this algorithm and it did not reveal
any issues. Also in this unit tests I measured percent of rollbacks when
concurrent updates have lots of conflicts: with 'try-lock' approach percent
of rollbacks is ~80%, with new algorithm is ~1% (but of course with
real-life test results can be different).

Does anyone see problems with this locking approach?

Re: Optimistic/serializable transactions implementation

Posted by Dmitriy Setrakyan <ds...@apache.org>.
On Fri, Oct 16, 2015 at 1:04 AM, Yakov Zhdanov <yz...@gridgain.com>
wrote:

> Agree with Sam here - we also reduce network calls since we don't need to
> notify for each failed tryLock, but only for cases when tx will definitely
> loose.
>

I also like this approach. Deadlock-free transactions is a HUGE feature and
I can't wait to blog about it. On top of that, it looks like it will be the
fastest transactional mode as well.

I think we should spend some time documenting the algorithm in detail, so
users will have precise understanding on how it works.


>
> Thanks!
> --
> Yakov Zhdanov, Director R&D
> *GridGain Systems*
> www.gridgain.com
>
> 2015-10-16 10:22 GMT+03:00 Semyon Boikov <sb...@gridgain.com>:
>
> > >
> > > The success ratio with ordered approach is naturally going to be
> better.
> > > However, I think the performance will suffer, because queues are
> > generally
> > > expensive. Have you tried comparing performance of the queue-based
> > approach
> > > vs. the try-lock one?
> >
> >
> > I did not add any new queues and just use existing per-entry list of mvcc
> > candidates, so ordered approach will definitely perform better than
> > try-lock.
> >
> > On Fri, Oct 16, 2015 at 4:10 AM, Dmitriy Setrakyan <
> dsetrakyan@apache.org>
> > wrote:
> >
> > > On Thu, Oct 15, 2015 at 12:00 AM, Semyon Boikov <sb...@gridgain.com>
> > > wrote:
> > >
> > > It seems there is better approach to resolve these conflicts to do not
> > fail
> > > > all conflicting transactions:
> > > > - we should order all transactions by some attribute (e.g. all
> > > transactions
> > > > already have unique version)
> > > > - transaction with greater order should always 'win' transaction with
> > > lower
> > > > order
> > > > - per-entry queue with waiting transactions should be sorted by this
> > > order
> > > > - when transaction tries to acquire entry lock it is added in waiting
> > > queue
> > > > if queue is empty or last waiting transaction have lower order,
> > otherwise
> > > > transaction fails
> > > >
> > > > With this approach transaction lock assignment is ordered and
> > > transactions
> > > > with lower order never wait for transactions with greater order, so
> > this
> > > > algorithm should not cause deadlocks.
> > > >
> > > > I also created unit test simulating this algorithm and it did not
> > reveal
> > > > any issues. Also in this unit tests I measured percent of rollbacks
> > when
> > > > concurrent updates have lots of conflicts: with 'try-lock' approach
> > > percent
> > > > of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> > > > real-life test results can be different).
> > > >
> > >
> > > The success ratio with ordered approach is naturally going to be
> better.
> > > However, I think the performance will suffer, because queues are
> > generally
> > > expensive. Have you tried comparing performance of the queue-based
> > approach
> > > vs. the try-lock one?
> > >
> >
>

Re: Optimistic/serializable transactions implementation

Posted by Yakov Zhdanov <yz...@gridgain.com>.
Agree with Sam here - we also reduce network calls since we don't need to
notify for each failed tryLock, but only for cases when tx will definitely
loose.

Thanks!
--
Yakov Zhdanov, Director R&D
*GridGain Systems*
www.gridgain.com

2015-10-16 10:22 GMT+03:00 Semyon Boikov <sb...@gridgain.com>:

> >
> > The success ratio with ordered approach is naturally going to be better.
> > However, I think the performance will suffer, because queues are
> generally
> > expensive. Have you tried comparing performance of the queue-based
> approach
> > vs. the try-lock one?
>
>
> I did not add any new queues and just use existing per-entry list of mvcc
> candidates, so ordered approach will definitely perform better than
> try-lock.
>
> On Fri, Oct 16, 2015 at 4:10 AM, Dmitriy Setrakyan <ds...@apache.org>
> wrote:
>
> > On Thu, Oct 15, 2015 at 12:00 AM, Semyon Boikov <sb...@gridgain.com>
> > wrote:
> >
> > It seems there is better approach to resolve these conflicts to do not
> fail
> > > all conflicting transactions:
> > > - we should order all transactions by some attribute (e.g. all
> > transactions
> > > already have unique version)
> > > - transaction with greater order should always 'win' transaction with
> > lower
> > > order
> > > - per-entry queue with waiting transactions should be sorted by this
> > order
> > > - when transaction tries to acquire entry lock it is added in waiting
> > queue
> > > if queue is empty or last waiting transaction have lower order,
> otherwise
> > > transaction fails
> > >
> > > With this approach transaction lock assignment is ordered and
> > transactions
> > > with lower order never wait for transactions with greater order, so
> this
> > > algorithm should not cause deadlocks.
> > >
> > > I also created unit test simulating this algorithm and it did not
> reveal
> > > any issues. Also in this unit tests I measured percent of rollbacks
> when
> > > concurrent updates have lots of conflicts: with 'try-lock' approach
> > percent
> > > of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> > > real-life test results can be different).
> > >
> >
> > The success ratio with ordered approach is naturally going to be better.
> > However, I think the performance will suffer, because queues are
> generally
> > expensive. Have you tried comparing performance of the queue-based
> approach
> > vs. the try-lock one?
> >
>

Re: Optimistic/serializable transactions implementation

Posted by Semyon Boikov <sb...@gridgain.com>.
>
> The success ratio with ordered approach is naturally going to be better.
> However, I think the performance will suffer, because queues are generally
> expensive. Have you tried comparing performance of the queue-based approach
> vs. the try-lock one?


I did not add any new queues and just use existing per-entry list of mvcc
candidates, so ordered approach will definitely perform better than
try-lock.

On Fri, Oct 16, 2015 at 4:10 AM, Dmitriy Setrakyan <ds...@apache.org>
wrote:

> On Thu, Oct 15, 2015 at 12:00 AM, Semyon Boikov <sb...@gridgain.com>
> wrote:
>
> It seems there is better approach to resolve these conflicts to do not fail
> > all conflicting transactions:
> > - we should order all transactions by some attribute (e.g. all
> transactions
> > already have unique version)
> > - transaction with greater order should always 'win' transaction with
> lower
> > order
> > - per-entry queue with waiting transactions should be sorted by this
> order
> > - when transaction tries to acquire entry lock it is added in waiting
> queue
> > if queue is empty or last waiting transaction have lower order, otherwise
> > transaction fails
> >
> > With this approach transaction lock assignment is ordered and
> transactions
> > with lower order never wait for transactions with greater order, so this
> > algorithm should not cause deadlocks.
> >
> > I also created unit test simulating this algorithm and it did not reveal
> > any issues. Also in this unit tests I measured percent of rollbacks when
> > concurrent updates have lots of conflicts: with 'try-lock' approach
> percent
> > of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> > real-life test results can be different).
> >
>
> The success ratio with ordered approach is naturally going to be better.
> However, I think the performance will suffer, because queues are generally
> expensive. Have you tried comparing performance of the queue-based approach
> vs. the try-lock one?
>

Re: Optimistic/serializable transactions implementation

Posted by Dmitriy Setrakyan <ds...@apache.org>.
On Thu, Oct 15, 2015 at 12:00 AM, Semyon Boikov <sb...@gridgain.com>
wrote:

It seems there is better approach to resolve these conflicts to do not fail
> all conflicting transactions:
> - we should order all transactions by some attribute (e.g. all transactions
> already have unique version)
> - transaction with greater order should always 'win' transaction with lower
> order
> - per-entry queue with waiting transactions should be sorted by this order
> - when transaction tries to acquire entry lock it is added in waiting queue
> if queue is empty or last waiting transaction have lower order, otherwise
> transaction fails
>
> With this approach transaction lock assignment is ordered and transactions
> with lower order never wait for transactions with greater order, so this
> algorithm should not cause deadlocks.
>
> I also created unit test simulating this algorithm and it did not reveal
> any issues. Also in this unit tests I measured percent of rollbacks when
> concurrent updates have lots of conflicts: with 'try-lock' approach percent
> of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> real-life test results can be different).
>

The success ratio with ordered approach is naturally going to be better.
However, I think the performance will suffer, because queues are generally
expensive. Have you tried comparing performance of the queue-based approach
vs. the try-lock one?

Re: Optimistic/serializable transactions implementation

Posted by Semyon Boikov <sb...@gridgain.com>.
Jira issue is https://issues.apache.org/jira/browse/IGNITE-1607, branch
ignite-1607

On Thu, Oct 15, 2015 at 10:58 AM, Alexey Kuznetsov <ak...@gridgain.com>
wrote:

> Semyon, could you please give a link to JIRA issue (if any) and what branch
> contains your code?
>
> Also it is not clear for me, how transaction order is assigned /
> calculated?
> If I start transaction t1 on none n1 and t2 on node n2, how it will be
> calculated?
>
> Thanks.
>
> On Thu, Oct 15, 2015 at 2:00 PM, Semyon Boikov <sb...@gridgain.com>
> wrote:
>
> > Hello,
> >
> > I'm working on new implementation for optimistic/serializable transaction
> > mode (current implementation is inefficient and have bugs). This mode is
> > supposed to be used when concurrent transactions do not update the same
> > keys, in this case transactions can be executed more efficiently, but if
> > concurrent transactions have read of write conflicts then
> > TransactionOptimisticException can be thrown and transaction should be
> > retried. Also with current transactions implementation user should order
> > updated keys, otherwise deadlock is possible, we want to remove this
> > requirement for optimistic/serializable mode.
> >
> > Issue with read/write conflicts can be detected if when read is performed
> > entry version is stored and then compared with current version when
> > transaction lock is acquired.
> >
> > Another issue is avoid deadlocks when transactions acquire keys in
> > different order.
> >
> > I created one possible solution using 'try-lock' approach: for each cache
> > entry we already have queue with current lock owner and transactions
> trying
> > to acquire entry lock.
> > When optimistic/serializable transaction tries to acquire entry lock it
> > checks that entry is not locked and there are no others transactions
> > waiting for lock entry, otherwise transaction fails with
> > TransactionOptimisticException. But this approach does not work well when
> > two transaction have lot of intersecting keys: it is possible that these
> > transaction will constantly conflict and will both constantly fail with
> > TransactionOptimisticException.
> >
> > It seems there is better approach to resolve these conflicts to do not
> fail
> > all conflicting transactions:
> > - we should order all transactions by some attribute (e.g. all
> transactions
> > already have unique version)
> > - transaction with greater order should always 'win' transaction with
> lower
> > order
> > - per-entry queue with waiting transactions should be sorted by this
> order
> > - when transaction tries to acquire entry lock it is added in waiting
> queue
> > if queue is empty or last waiting transaction have lower order, otherwise
> > transaction fails
> >
> > With this approach transaction lock assignment is ordered and
> transactions
> > with lower order never wait for transactions with greater order, so this
> > algorithm should not cause deadlocks.
> >
> > I also created unit test simulating this algorithm and it did not reveal
> > any issues. Also in this unit tests I measured percent of rollbacks when
> > concurrent updates have lots of conflicts: with 'try-lock' approach
> percent
> > of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> > real-life test results can be different).
> >
> > Does anyone see problems with this locking approach?
> >
>
>
>
> --
> Alexey Kuznetsov
> GridGain Systems
> www.gridgain.com
>

Re: Optimistic/serializable transactions implementation

Posted by Alexey Goncharuk <al...@gmail.com>.
Strictly speaking there is a small chance of globalTime clash, but
something like (globalTime, nodeOrder) should do.

2015-10-15 11:43 GMT+03:00 Vladimir Ozerov <vo...@gridgain.com>:

> Looks like ordering semantics is the most critical part of this algorithm
> :-) If we are really bothered with possible starvation, then neither
> topology version, nor local order, node ID or keys count could help us. Any
> combination of them allows for olders TXs to be postponed by newer.
> May be GridCacheVersion.globalTime will do the trick?
>
> On Thu, Oct 15, 2015 at 11:32 AM, Semyon Boikov <sb...@gridgain.com>
> wrote:
>
> > We did not decided yet exactly by which attribute transactions should be
> > ordered, but logically it is better when wins older transaction or
> > transaction having more keys.
> >
> > On Thu, Oct 15, 2015 at 11:18 AM, Alexey Kuznetsov <
> > akuznetsov@gridgain.com>
> > wrote:
> >
> > > Just one more question:
> > >
> > > "- transaction with greater order should always 'win' transaction with
> > > lower order"
> > >
> > > Greater order means "younger"?
> > >
> > > If it so, why should younger transactions win? Why not older?
> > >
> > > Or user will have possibility to configure this aspect of conflict
> > > resolution?
> > >
> > > On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
> > > alexey.goncharuk@gmail.com> wrote:
> > >
> > > > 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <akuznetsov@gridgain.com
> >:
> > > > >
> > > > > Also it is not clear for me, how transaction order is assigned /
> > > > > calculated?
> > > > > If I start transaction t1 on none n1 and t2 on node n2, how it will
> > be
> > > > > calculated?
> > > > >
> > > > I believe that we can utilize nearXidVersion for this ordering (or
> some
> > > > sort of it's modification). Since cache version contains local order,
> > > > topology version and node ID and also is comparable, it is guaranteed
> > > that
> > > > nearXidVersion is always unique and there is always an unambiguous
> > order
> > > > between any two Xid versions.
> > > >
> > >
> > >
> > >
> > > --
> > > Alexey Kuznetsov
> > > GridGain Systems
> > > www.gridgain.com
> > >
> >
>

Re: Optimistic/serializable transactions implementation

Posted by Vladimir Ozerov <vo...@gridgain.com>.
Looks like ordering semantics is the most critical part of this algorithm
:-) If we are really bothered with possible starvation, then neither
topology version, nor local order, node ID or keys count could help us. Any
combination of them allows for olders TXs to be postponed by newer.
May be GridCacheVersion.globalTime will do the trick?

On Thu, Oct 15, 2015 at 11:32 AM, Semyon Boikov <sb...@gridgain.com>
wrote:

> We did not decided yet exactly by which attribute transactions should be
> ordered, but logically it is better when wins older transaction or
> transaction having more keys.
>
> On Thu, Oct 15, 2015 at 11:18 AM, Alexey Kuznetsov <
> akuznetsov@gridgain.com>
> wrote:
>
> > Just one more question:
> >
> > "- transaction with greater order should always 'win' transaction with
> > lower order"
> >
> > Greater order means "younger"?
> >
> > If it so, why should younger transactions win? Why not older?
> >
> > Or user will have possibility to configure this aspect of conflict
> > resolution?
> >
> > On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
> > alexey.goncharuk@gmail.com> wrote:
> >
> > > 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
> > > >
> > > > Also it is not clear for me, how transaction order is assigned /
> > > > calculated?
> > > > If I start transaction t1 on none n1 and t2 on node n2, how it will
> be
> > > > calculated?
> > > >
> > > I believe that we can utilize nearXidVersion for this ordering (or some
> > > sort of it's modification). Since cache version contains local order,
> > > topology version and node ID and also is comparable, it is guaranteed
> > that
> > > nearXidVersion is always unique and there is always an unambiguous
> order
> > > between any two Xid versions.
> > >
> >
> >
> >
> > --
> > Alexey Kuznetsov
> > GridGain Systems
> > www.gridgain.com
> >
>

Re: Optimistic/serializable transactions implementation

Posted by Dmitriy Setrakyan <ds...@apache.org>.
On Thu, Oct 15, 2015 at 1:32 AM, Semyon Boikov <sb...@gridgain.com> wrote:

> We did not decided yet exactly by which attribute transactions should be
> ordered, but logically it is better when wins older transaction or
> transaction having more keys.


I think the ordering should have first-come-first-serve semantics, so the
older transactions should win.

Re: Optimistic/serializable transactions implementation

Posted by Semyon Boikov <sb...@gridgain.com>.
We did not decided yet exactly by which attribute transactions should be
ordered, but logically it is better when wins older transaction or
transaction having more keys.

On Thu, Oct 15, 2015 at 11:18 AM, Alexey Kuznetsov <ak...@gridgain.com>
wrote:

> Just one more question:
>
> "- transaction with greater order should always 'win' transaction with
> lower order"
>
> Greater order means "younger"?
>
> If it so, why should younger transactions win? Why not older?
>
> Or user will have possibility to configure this aspect of conflict
> resolution?
>
> On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
> alexey.goncharuk@gmail.com> wrote:
>
> > 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
> > >
> > > Also it is not clear for me, how transaction order is assigned /
> > > calculated?
> > > If I start transaction t1 on none n1 and t2 on node n2, how it will be
> > > calculated?
> > >
> > I believe that we can utilize nearXidVersion for this ordering (or some
> > sort of it's modification). Since cache version contains local order,
> > topology version and node ID and also is comparable, it is guaranteed
> that
> > nearXidVersion is always unique and there is always an unambiguous order
> > between any two Xid versions.
> >
>
>
>
> --
> Alexey Kuznetsov
> GridGain Systems
> www.gridgain.com
>

Re: Optimistic/serializable transactions implementation

Posted by Semyon Boikov <sb...@gridgain.com>.
>
> is starvation possible here?
> E.g. there are two nodes with GUIDs A and B. What will happen in the
> following case:
> 1) TX (A, 1) started and locked the key;
> 2) TX (B, 1) started and waiting for lock;
> 3) TX (A, 2) started and waiting for lock;
> 4) TX (A, 1) released the lock.
> 5) Who wins now? If this is (A, 2), then lock acquisition by the node B can
> be postponed indefinitely.


I assume transactions are ordered as TX(A, 1), TX(A, 2), TX(B, 1). In this
case when TX(A, 2) tries to get lock it sees that there is already
waiting TX(B,
1) with greater order and it fails. So TX(A, 1), TX(B, 1) will finish
succesfully, TX(A,2) should be retried.

On Thu, Oct 15, 2015 at 11:31 AM, Vladimir Ozerov <vo...@gridgain.com>
wrote:

> is starvation possible here?
>
> E.g. there are two nodes with GUIDs A and B. What will happen in the
> following case:
> 1) TX (A, 1) started and locked the key;
> 2) TX (B, 1) started and waiting for lock;
> 3) TX (A, 2) started and waiting for lock;
> 4) TX (A, 1) released the lock.
> 5) Who wins now? If this is (A, 2), then lock acquisition by the node B can
> be postponed indefinitely.
>
> On Thu, Oct 15, 2015 at 11:18 AM, Alexey Kuznetsov <
> akuznetsov@gridgain.com>
> wrote:
>
> > Just one more question:
> >
> > "- transaction with greater order should always 'win' transaction with
> > lower order"
> >
> > Greater order means "younger"?
> >
> > If it so, why should younger transactions win? Why not older?
> >
> > Or user will have possibility to configure this aspect of conflict
> > resolution?
> >
> > On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
> > alexey.goncharuk@gmail.com> wrote:
> >
> > > 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
> > > >
> > > > Also it is not clear for me, how transaction order is assigned /
> > > > calculated?
> > > > If I start transaction t1 on none n1 and t2 on node n2, how it will
> be
> > > > calculated?
> > > >
> > > I believe that we can utilize nearXidVersion for this ordering (or some
> > > sort of it's modification). Since cache version contains local order,
> > > topology version and node ID and also is comparable, it is guaranteed
> > that
> > > nearXidVersion is always unique and there is always an unambiguous
> order
> > > between any two Xid versions.
> > >
> >
> >
> >
> > --
> > Alexey Kuznetsov
> > GridGain Systems
> > www.gridgain.com
> >
>

Re: Optimistic/serializable transactions implementation

Posted by Vladimir Ozerov <vo...@gridgain.com>.
is starvation possible here?

E.g. there are two nodes with GUIDs A and B. What will happen in the
following case:
1) TX (A, 1) started and locked the key;
2) TX (B, 1) started and waiting for lock;
3) TX (A, 2) started and waiting for lock;
4) TX (A, 1) released the lock.
5) Who wins now? If this is (A, 2), then lock acquisition by the node B can
be postponed indefinitely.

On Thu, Oct 15, 2015 at 11:18 AM, Alexey Kuznetsov <ak...@gridgain.com>
wrote:

> Just one more question:
>
> "- transaction with greater order should always 'win' transaction with
> lower order"
>
> Greater order means "younger"?
>
> If it so, why should younger transactions win? Why not older?
>
> Or user will have possibility to configure this aspect of conflict
> resolution?
>
> On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
> alexey.goncharuk@gmail.com> wrote:
>
> > 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
> > >
> > > Also it is not clear for me, how transaction order is assigned /
> > > calculated?
> > > If I start transaction t1 on none n1 and t2 on node n2, how it will be
> > > calculated?
> > >
> > I believe that we can utilize nearXidVersion for this ordering (or some
> > sort of it's modification). Since cache version contains local order,
> > topology version and node ID and also is comparable, it is guaranteed
> that
> > nearXidVersion is always unique and there is always an unambiguous order
> > between any two Xid versions.
> >
>
>
>
> --
> Alexey Kuznetsov
> GridGain Systems
> www.gridgain.com
>

Re: Optimistic/serializable transactions implementation

Posted by Alexey Kuznetsov <ak...@gridgain.com>.
Just one more question:

"- transaction with greater order should always 'win' transaction with
lower order"

Greater order means "younger"?

If it so, why should younger transactions win? Why not older?

Or user will have possibility to configure this aspect of conflict
resolution?

On Thu, Oct 15, 2015 at 3:07 PM, Alexey Goncharuk <
alexey.goncharuk@gmail.com> wrote:

> 2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
> >
> > Also it is not clear for me, how transaction order is assigned /
> > calculated?
> > If I start transaction t1 on none n1 and t2 on node n2, how it will be
> > calculated?
> >
> I believe that we can utilize nearXidVersion for this ordering (or some
> sort of it's modification). Since cache version contains local order,
> topology version and node ID and also is comparable, it is guaranteed that
> nearXidVersion is always unique and there is always an unambiguous order
> between any two Xid versions.
>



-- 
Alexey Kuznetsov
GridGain Systems
www.gridgain.com

Re: Optimistic/serializable transactions implementation

Posted by Alexey Goncharuk <al...@gmail.com>.
2015-10-15 10:58 GMT+03:00 Alexey Kuznetsov <ak...@gridgain.com>:
>
> Also it is not clear for me, how transaction order is assigned /
> calculated?
> If I start transaction t1 on none n1 and t2 on node n2, how it will be
> calculated?
>
I believe that we can utilize nearXidVersion for this ordering (or some
sort of it's modification). Since cache version contains local order,
topology version and node ID and also is comparable, it is guaranteed that
nearXidVersion is always unique and there is always an unambiguous order
between any two Xid versions.

Re: Optimistic/serializable transactions implementation

Posted by Alexey Kuznetsov <ak...@gridgain.com>.
Semyon, could you please give a link to JIRA issue (if any) and what branch
contains your code?

Also it is not clear for me, how transaction order is assigned / calculated?
If I start transaction t1 on none n1 and t2 on node n2, how it will be
calculated?

Thanks.

On Thu, Oct 15, 2015 at 2:00 PM, Semyon Boikov <sb...@gridgain.com> wrote:

> Hello,
>
> I'm working on new implementation for optimistic/serializable transaction
> mode (current implementation is inefficient and have bugs). This mode is
> supposed to be used when concurrent transactions do not update the same
> keys, in this case transactions can be executed more efficiently, but if
> concurrent transactions have read of write conflicts then
> TransactionOptimisticException can be thrown and transaction should be
> retried. Also with current transactions implementation user should order
> updated keys, otherwise deadlock is possible, we want to remove this
> requirement for optimistic/serializable mode.
>
> Issue with read/write conflicts can be detected if when read is performed
> entry version is stored and then compared with current version when
> transaction lock is acquired.
>
> Another issue is avoid deadlocks when transactions acquire keys in
> different order.
>
> I created one possible solution using 'try-lock' approach: for each cache
> entry we already have queue with current lock owner and transactions trying
> to acquire entry lock.
> When optimistic/serializable transaction tries to acquire entry lock it
> checks that entry is not locked and there are no others transactions
> waiting for lock entry, otherwise transaction fails with
> TransactionOptimisticException. But this approach does not work well when
> two transaction have lot of intersecting keys: it is possible that these
> transaction will constantly conflict and will both constantly fail with
> TransactionOptimisticException.
>
> It seems there is better approach to resolve these conflicts to do not fail
> all conflicting transactions:
> - we should order all transactions by some attribute (e.g. all transactions
> already have unique version)
> - transaction with greater order should always 'win' transaction with lower
> order
> - per-entry queue with waiting transactions should be sorted by this order
> - when transaction tries to acquire entry lock it is added in waiting queue
> if queue is empty or last waiting transaction have lower order, otherwise
> transaction fails
>
> With this approach transaction lock assignment is ordered and transactions
> with lower order never wait for transactions with greater order, so this
> algorithm should not cause deadlocks.
>
> I also created unit test simulating this algorithm and it did not reveal
> any issues. Also in this unit tests I measured percent of rollbacks when
> concurrent updates have lots of conflicts: with 'try-lock' approach percent
> of rollbacks is ~80%, with new algorithm is ~1% (but of course with
> real-life test results can be different).
>
> Does anyone see problems with this locking approach?
>



-- 
Alexey Kuznetsov
GridGain Systems
www.gridgain.com