You are viewing a plain text version of this content. The canonical link for it is here.
Posted to user@kudu.apache.org by Xiaokai Wang <xi...@live.com> on 2018/09/18 04:12:22 UTC

Locks are acquired to cost much time in transactions

Moved here from JIRA.


Hi guys, I met a problem about the keys locks that almost impacts the service normal writing.


As we all know, a transaction which get all row_key locks will go on next step in kudu. Everything looks good, if keys are not concurrent updated. But when keys are updated by more than one client at the same time, locks are acquired to wait much time. The cases are often in my product environment. Does anybody meet the problem? Has any good ideal for this?


In my way, I want to try to abandon keys locks, instead using *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial and ordered. Dose this work?


Hope to get your advice. Thanks.


-----
Regards,
Xiaokai

Re: Locks are acquired to cost much time in transactions

Posted by Xiaokai Wang <xi...@live.com>.
Thanks your reply, Mike.

In our scenario, some tables have a lot of data to be updated at the same time. The current implementation of KUDU, if TXN(transaction) can not get all the ops lock it will wait for 1s to block the entire Tablet processing, which will greatly reduce the write performance of the Tablet.Reduce the size of the batch, TXN may still have the same row key, and can not solve this problem.


In my way, I want to change the work way of 'APPLY' phase, instead of putting TXN to apply_pool_ queue, I will put each key-op(splitting TXN to ops) to apply_pool_token_ queue. This will guarantee the same key's op will be putted to the same thread. When ops belonging to the same TXN execute over, then send rpc response and write CommitMsg to WAL. In this way, I can abandon the keys locks.

Working follow chart just like this below:
[op-key_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn in turn.

2. leader send replicated msg to follower, sending them together or independent. T1, T2, T3 will be ordered to execute by follower and send them back by together or independent in turn. raft_pool_token_ guarantee T1, T2 T3 be ordered to apply.

3. Splitting Tx to ops, each op is hashed to the queue of apply_pool_token_, key's op is order to execute as TXN turn.

--------

Kudu origin work way below:
[txn_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn orderly, they will acquire locks in 'PREPARE' phase in turn.

2. leader send replicated msg to follower, sending them together or independent. T1, T2, T3 will be ordered to execute by follower and send them back by together or independent in turn. raft_pool_token_ guarantee T1, T2 T3 be ordered to apply.

3. T1, T2, T3 are putted to the queue of apply_pool_, order or concurrented to be executed, releasing locks.

-----------

Contrast to origin, the keys locks are abandon, this can guarantee kudu throughput more smoothly. What do you think? Hope to get your advice, kudu users.

Thanks.
xiaokai.


---------- Forwarded message ---------
From: Mike Percy <mp...@apache.org>>
Date: 2018年9月19日周三 上午8:29
Subject: Re: Locks are acquired to cost much time in transactions
To: <us...@kudu.apache.org>>


Why do you think you are spending a lot of time contending on row locks?

Have you tried configuring your clients to send smaller batches? This may decrease throughput on a per-client basis but will likely improve latency and reduce the likelihood of row lock contention.

If you are really spending most of your time contending on row locks then you will likely run into more fundamental performance issues trying to scale your writes, since Kudu's MVCC implementation effectively stores a linked list of updates to a given cell until compaction occurs. See https://github.com/apache/kudu/blob/master/docs/design-docs/tablet.md#historical-mvcc-in-diskrowsets for more information about the on-disk design.

If you accumulate too many uncompacted mutations against a given row, reading the latest value for that row at scan time will be slow because it has to do a lot of work at read time.

Mike

On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com>> wrote:
Moved here from JIRA.


Hi guys, I met a problem about the keys locks that almost impacts the service normal writing.


As we all know, a transaction which get all row_key locks will go on next step in kudu. Everything looks good, if keys are not concurrent updated. But when keys are updated by more than one client at the same time, locks are acquired to wait much time. The cases are often in my product environment. Does anybody meet the problem? Has any good ideal for this?


In my way, I want to try to abandon keys locks, instead using *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial and ordered. Dose this work?


Hope to get your advice. Thanks.


-----
Regards,
Xiaokai

Re: Locks are acquired to cost much time in transactions

Posted by Hao Hao <ha...@cloudera.com>.
Hi Xiaokai,

Thanks a lot for the explanation! Yeah, it would be good to file a jira and
add more detail information in a design doc
or alike for other people to comment.

Another thing you might want to note is how transaction abort will be
handled in your new proposal.

Best,
Hao

On Wed, Sep 26, 2018 at 10:16 AM Xiaokai Wang <xi...@live.com> wrote:

> Hi Hao,
>
> Thanks to your reply, hao.
>
>  you are proposing to use key->ops hash to apply the write transaction instead
> of serializing the operations in PREPARE phrase with a lock?
>
> Changing thread mode from concurrent to serial in APPLY phase. Splitting
> TXN to ops, enqueuing the op to apply_pool token based on
> hash(primary-key), then the op(primary key) can be applied in turn.
>
> 1) what the time stamp would be if there is no longer a lock during
> PREPARE phrase, how to ensure
>    the following scenario happens in order?
>
>    - T1 deletes the row (key1, key2, value1)
>    - T2 upserts the row (key1, key2, value2)
>
>
> It is ordered in PREPARE phase, so timestamp will be increased as TXN
> order, and also op-index, raft protocol guarantee TXN are orderly submitted
> to apply_pool_ by op-index. We just keep keys are ordered to apply in APPLY
> phase after abandoning row-key lock, as the plan says.
>
> 2) what happens if any of the operations happens to update the keys? e.g
>
>    - T1 updates the row (key1, key2, value1) to (key3, key2, value2)
>    - T1 updates the row (key3, key2, value3) to (key1, key2, value1)
>
>
> Like to be described above, it is ordered to apply by key.
>
> I will submit a JIRA and attach the design doc later. Hope and please you
> keep a eye about that.
>
> Thanks.
>
> -----
> Regards,
> Xiaokai
>
> ------------------------------
> *From:* Hao Hao <ha...@cloudera.com>
> *Sent:* Monday, September 24, 2018 11:18 PM
> *To:* user@kudu.apache.org
> *Subject:* Re: Locks are acquired to cost much time in transactions
>
> Hi Xiaokai,
>
> If I understand you correctly, you are proposing to use key->ops hash to
> apply the write transaction
> instead of serializing the operations in PREPARE phrase with a lock? Here
> are some questions I
> have with this approach,
>
> 1) what the time stamp would be if there is no longer a lock during
> PREPARE phrase, how to ensure
>    the following scenario happens in order?
>
>    - T1 deletes the row (key1, key2, value1)
>    - T2 upserts the row (key1, key2, value2)
>
> 2) what happens if any of the operations happens to update the keys? e.g
>
>    - T1 updates the row (key1, key2, value1) to (key3, key2, value2)
>    - T1 updates the row (key3, key2, value3) to (key1, key2, value1)
>
> Both of the above scenarios don't seem to have a deterministic behavior.
>
> Best,
> Hao
>
>
> On Fri, Sep 21, 2018 at 2:29 AM, Xiaokai Wang <xi...@gmail.com>
> wrote:
>
> Thanks your reply, Mike.
>
> In our scenario, some tables have a lot of data to be updated at the same
> time. The current implementation of KUDU, if TXN(transaction) can not get
> all the ops lock it will wait for 1s to block the entire Tablet processing, which
> will greatly reduce the write performance of the Tablet.Reduce the size
> of the batch, TXN may still have the same row key, and can not solve this
> problem.
>
>
> In my way, I want to change the work way of 'APPLY' phase, instead of
> putting TXN to apply_pool_ queue, I will put each key-op(splitting TXN to
> ops) to apply_pool_token_ queue. This will guarantee the same key's op
> will be putted to the same thread. When ops belonging to the same TXN
> execute over, then send rpc response and write CommitMsg to WAL. In this
> way, I can abandon the keys locks.
>
> Working follow chart just like this below:
> [image: op-key_concurrent.png]
> 1. client send T1, T2, T3 to tserver, tserver will handle each txn in turn.
>
> 2. leader send replicated msg to follower, sending them together or
> independent. T1, T2, T3 will be ordered to execute by follower and send
> them back by together or independent in turn. raft_pool_token_ guarantee
> T1, T2 T3 be ordered to apply.
>
> 3. Splitting Tx to ops, each op is hashed to the queue of
> apply_pool_token_, key's op is order to execute as TXN turn.
>
> --------
>
> Kudu origin work way below:
> [image: txn_concurrent.png]
> 1. client send T1, T2, T3 to tserver, tserver will handle each txn
> orderly, they will acquire locks in 'PREPARE' phase in turn.
>
> 2. leader send replicated msg to follower, sending them together or
> independent. T1, T2, T3 will be ordered to execute by follower and send
> them back by together or independent in turn. raft_pool_token_ guarantee
> T1, T2 T3 be ordered to apply.
>
> 3. T1, T2, T3 are putted to the queue of apply_pool_, order or
> concurrented to be executed, releasing locks.
>
> -----------
>
> Contrast to origin, the keys locks are abandon, this can guarantee kudu
> throughput more smoothly. What do you think? Hope to get your advice,
> kudu users.
>
> Thanks.
> xiaokai.
>
>
> ---------- Forwarded message ---------
> From: *Mike Percy* <mp...@apache.org>
> Date: 2018年9月19日周三 上午8:29
> Subject: Re: Locks are acquired to cost much time in transactions
> To: <us...@kudu.apache.org>
>
>
> Why do you think you are spending a lot of time contending on row locks?
>
> Have you tried configuring your clients to send smaller batches? This may
> decrease throughput on a per-client basis but will likely improve latency
> and reduce the likelihood of row lock contention.
>
> If you are really spending most of your time contending on row locks then
> you will likely run into more fundamental performance issues trying to
> scale your writes, since Kudu's MVCC implementation effectively stores a
> linked list of updates to a given cell until compaction occurs. See
> https://github.com/apache/kudu/blob/master/docs/design-docs/tablet.md#historical-mvcc-in-diskrowsets
> for more information about the on-disk design.
>
> If you accumulate too many uncompacted mutations against a given row,
> reading the latest value for that row at scan time will be slow because it
> has to do a lot of work at read time.
>
> Mike
>
> On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com>
> wrote:
>
> Moved here from JIRA.
>
> Hi guys, I met a problem about the keys locks that almost impacts the
> service normal writing.
>
>
> As we all know, a transaction which get all row_key locks will go on next
> step in kudu. Everything looks good, if keys are not concurrent updated.
> But when keys are updated by more than one client at the same time, locks
> are acquired to wait much time. The cases are often in my product
> environment. Does anybody meet the problem? Has any good ideal for this?
>
>
> In my way, I want to try to abandon keys locks, instead using
> *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial
> and ordered. Dose this work?
>
>
> Hope to get your advice. Thanks.
>
>
> -----
> Regards,
> Xiaokai
>
>
>

Re: Locks are acquired to cost much time in transactions

Posted by Xiaokai Wang <xi...@live.com>.
Hi Hao,

Thanks to your reply, hao.

 you are proposing to use key->ops hash to apply the write transaction instead of serializing the operations in PREPARE phrase with a lock?
Changing thread mode from concurrent to serial in APPLY phase. Splitting TXN to ops, enqueuing the op to apply_pool token based on hash(primary-key), then the op(primary key) can be applied in turn.

1) what the time stamp would be if there is no longer a lock during PREPARE phrase, how to ensure
   the following scenario happens in order?

  *   T1 deletes the row (key1, key2, value1)
  *   T2 upserts the row (key1, key2, value2)

It is ordered in PREPARE phase, so timestamp will be increased as TXN order, and also op-index, raft protocol guarantee TXN are orderly submitted to apply_pool_ by op-index. We just keep keys are ordered to apply in APPLY phase after abandoning row-key lock, as the plan says.

2) what happens if any of the operations happens to update the keys? e.g

  *   T1 updates the row (key1, key2, value1) to (key3, key2, value2)
  *   T1 updates the row (key3, key2, value3) to (key1, key2, value1)

Like to be described above, it is ordered to apply by key.

I will submit a JIRA and attach the design doc later. Hope and please you keep a eye about that.

Thanks.

-----
Regards,
Xiaokai

________________________________
From: Hao Hao <ha...@cloudera.com>
Sent: Monday, September 24, 2018 11:18 PM
To: user@kudu.apache.org
Subject: Re: Locks are acquired to cost much time in transactions

Hi Xiaokai,

If I understand you correctly, you are proposing to use key->ops hash to apply the write transaction
instead of serializing the operations in PREPARE phrase with a lock? Here are some questions I
have with this approach,

1) what the time stamp would be if there is no longer a lock during PREPARE phrase, how to ensure
   the following scenario happens in order?

  *   T1 deletes the row (key1, key2, value1)
  *   T2 upserts the row (key1, key2, value2)

2) what happens if any of the operations happens to update the keys? e.g

  *   T1 updates the row (key1, key2, value1) to (key3, key2, value2)
  *   T1 updates the row (key3, key2, value3) to (key1, key2, value1)

Both of the above scenarios don't seem to have a deterministic behavior.

Best,
Hao


On Fri, Sep 21, 2018 at 2:29 AM, Xiaokai Wang <xi...@gmail.com>> wrote:
Thanks your reply, Mike.


In our scenario, some tables have a lot of data to be updated at the same time. The current implementation of KUDU, if TXN(transaction) can not get all the ops lock it will wait for 1s to block the entire Tablet processing, which will greatly reduce the write performance of the Tablet.Reduce the size of the batch, TXN may still have the same row key, and can not solve this problem.


In my way, I want to change the work way of 'APPLY' phase, instead of putting TXN to apply_pool_ queue, I will put each key-op(splitting TXN to ops) to apply_pool_token_ queue. This will guarantee the same key's op will be putted to the same thread. When ops belonging to the same TXN execute over, then send rpc response and write CommitMsg to WAL. In this way, I can abandon the keys locks.

Working follow chart just like this below:
[op-key_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn in turn.

2. leader send replicated msg to follower, sending them together or independent. T1, T2, T3 will be ordered to execute by follower and send them back by together or independent in turn. raft_pool_token_ guarantee T1, T2 T3 be ordered to apply.

3. Splitting Tx to ops, each op is hashed to the queue of apply_pool_token_, key's op is order to execute as TXN turn.

--------

Kudu origin work way below:
[txn_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn orderly, they will acquire locks in 'PREPARE' phase in turn.

2. leader send replicated msg to follower, sending them together or independent. T1, T2, T3 will be ordered to execute by follower and send them back by together or independent in turn. raft_pool_token_ guarantee T1, T2 T3 be ordered to apply.

3. T1, T2, T3 are putted to the queue of apply_pool_, order or concurrented to be executed, releasing locks.

-----------

Contrast to origin, the keys locks are abandon, this can guarantee kudu throughput more smoothly. What do you think? Hope to get your advice, kudu users.

Thanks.
xiaokai.


---------- Forwarded message ---------
From: Mike Percy <mp...@apache.org>>
Date: 2018年9月19日周三 上午8:29
Subject: Re: Locks are acquired to cost much time in transactions
To: <us...@kudu.apache.org>>


Why do you think you are spending a lot of time contending on row locks?

Have you tried configuring your clients to send smaller batches? This may decrease throughput on a per-client basis but will likely improve latency and reduce the likelihood of row lock contention.

If you are really spending most of your time contending on row locks then you will likely run into more fundamental performance issues trying to scale your writes, since Kudu's MVCC implementation effectively stores a linked list of updates to a given cell until compaction occurs. See https://github.com/apache/kudu/blob/master/docs/design-docs/tablet.md#historical-mvcc-in-diskrowsets for more information about the on-disk design.

If you accumulate too many uncompacted mutations against a given row, reading the latest value for that row at scan time will be slow because it has to do a lot of work at read time.

Mike

On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com>> wrote:
Moved here from JIRA.


Hi guys, I met a problem about the keys locks that almost impacts the service normal writing.


As we all know, a transaction which get all row_key locks will go on next step in kudu. Everything looks good, if keys are not concurrent updated. But when keys are updated by more than one client at the same time, locks are acquired to wait much time. The cases are often in my product environment. Does anybody meet the problem? Has any good ideal for this?


In my way, I want to try to abandon keys locks, instead using *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial and ordered. Dose this work?


Hope to get your advice. Thanks.


-----
Regards,
Xiaokai


Re: Locks are acquired to cost much time in transactions

Posted by Hao Hao <ha...@cloudera.com>.
Hi Xiaokai,

If I understand you correctly, you are proposing to use key->ops hash to
apply the write transaction
instead of serializing the operations in PREPARE phrase with a lock? Here
are some questions I
have with this approach,

1) what the time stamp would be if there is no longer a lock during PREPARE
phrase, how to ensure
   the following scenario happens in order?

   - T1 deletes the row (key1, key2, value1)
   - T2 upserts the row (key1, key2, value2)

2) what happens if any of the operations happens to update the keys? e.g

   - T1 updates the row (key1, key2, value1) to (key3, key2, value2)
   - T1 updates the row (key3, key2, value3) to (key1, key2, value1)

Both of the above scenarios don't seem to have a deterministic behavior.

Best,
Hao


On Fri, Sep 21, 2018 at 2:29 AM, Xiaokai Wang <xi...@gmail.com>
wrote:

> Thanks your reply, Mike.
>
> In our scenario, some tables have a lot of data to be updated at the same
> time. The current implementation of KUDU, if TXN(transaction) can not get
> all the ops lock it will wait for 1s to block the entire Tablet processing, which
> will greatly reduce the write performance of the Tablet.Reduce the size
> of the batch, TXN may still have the same row key, and can not solve this
> problem.
>
>
> In my way, I want to change the work way of 'APPLY' phase, instead of
> putting TXN to apply_pool_ queue, I will put each key-op(splitting TXN to
> ops) to apply_pool_token_ queue. This will guarantee the same key's op
> will be putted to the same thread. When ops belonging to the same TXN
> execute over, then send rpc response and write CommitMsg to WAL. In this
> way, I can abandon the keys locks.
>
> Working follow chart just like this below:
> [image: op-key_concurrent.png]
> 1. client send T1, T2, T3 to tserver, tserver will handle each txn in turn.
>
> 2. leader send replicated msg to follower, sending them together or
> independent. T1, T2, T3 will be ordered to execute by follower and send
> them back by together or independent in turn. raft_pool_token_ guarantee
> T1, T2 T3 be ordered to apply.
>
> 3. Splitting Tx to ops, each op is hashed to the queue of
> apply_pool_token_, key's op is order to execute as TXN turn.
>
> --------
>
> Kudu origin work way below:
> [image: txn_concurrent.png]
> 1. client send T1, T2, T3 to tserver, tserver will handle each txn
> orderly, they will acquire locks in 'PREPARE' phase in turn.
>
> 2. leader send replicated msg to follower, sending them together or
> independent. T1, T2, T3 will be ordered to execute by follower and send
> them back by together or independent in turn. raft_pool_token_ guarantee
> T1, T2 T3 be ordered to apply.
>
> 3. T1, T2, T3 are putted to the queue of apply_pool_, order or
> concurrented to be executed, releasing locks.
>
> -----------
>
> Contrast to origin, the keys locks are abandon, this can guarantee kudu
> throughput more smoothly. What do you think? Hope to get your advice,
> kudu users.
>
> Thanks.
> xiaokai.
>
>
> ---------- Forwarded message ---------
>> From: Mike Percy <mp...@apache.org>
>> Date: 2018年9月19日周三 上午8:29
>> Subject: Re: Locks are acquired to cost much time in transactions
>> To: <us...@kudu.apache.org>
>>
>>
>> Why do you think you are spending a lot of time contending on row locks?
>>
>> Have you tried configuring your clients to send smaller batches? This may
>> decrease throughput on a per-client basis but will likely improve latency
>> and reduce the likelihood of row lock contention.
>>
>> If you are really spending most of your time contending on row locks then
>> you will likely run into more fundamental performance issues trying to
>> scale your writes, since Kudu's MVCC implementation effectively stores a
>> linked list of updates to a given cell until compaction occurs. See
>> https://github.com/apache/kudu/blob/master/docs/design-
>> docs/tablet.md#historical-mvcc-in-diskrowsets for more information about
>> the on-disk design.
>>
>> If you accumulate too many uncompacted mutations against a given row,
>> reading the latest value for that row at scan time will be slow because it
>> has to do a lot of work at read time.
>>
>> Mike
>>
>> On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com>
>> wrote:
>>
>>> Moved here from JIRA.
>>>
>>> Hi guys, I met a problem about the keys locks that almost impacts the
>>> service normal writing.
>>>
>>>
>>> As we all know, a transaction which get all row_key locks will go on
>>> next step in kudu. Everything looks good, if keys are not concurrent
>>> updated. But when keys are updated by more than one client at the same time
>>> , locks are acquired to wait much time. The cases are often in my
>>> product environment. Does anybody meet the problem? Has any good ideal for
>>> this?
>>>
>>>
>>> In my way, I want to try to abandon keys locks, instead using
>>> *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial
>>> and ordered. Dose this work?
>>>
>>>
>>> Hope to get your advice. Thanks.
>>>
>>>
>>> -----
>>> Regards,
>>> Xiaokai
>>>
>>

Re: Locks are acquired to cost much time in transactions

Posted by Xiaokai Wang <xi...@gmail.com>.
Thanks your reply, Mike.

In our scenario, some tables have a lot of data to be updated at the same
time. The current implementation of KUDU, if TXN(transaction) can not get
all the ops lock it will wait for 1s to block the entire Tablet
processing, which
will greatly reduce the write performance of the Tablet.Reduce the size of
the batch, TXN may still have the same row key, and can not solve this
problem.


In my way, I want to change the work way of 'APPLY' phase, instead of
putting TXN to apply_pool_ queue, I will put each key-op(splitting TXN to
ops) to apply_pool_token_ queue. This will guarantee the same key's op will
be putted to the same thread. When ops belonging to the same TXN execute
over, then send rpc response and write CommitMsg to WAL. In this way, I can
abandon the keys locks.

Working follow chart just like this below:
[image: op-key_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn in turn.

2. leader send replicated msg to follower, sending them together or
independent. T1, T2, T3 will be ordered to execute by follower and send
them back by together or independent in turn. raft_pool_token_ guarantee
T1, T2 T3 be ordered to apply.

3. Splitting Tx to ops, each op is hashed to the queue of
apply_pool_token_, key's op is order to execute as TXN turn.

--------

Kudu origin work way below:
[image: txn_concurrent.png]
1. client send T1, T2, T3 to tserver, tserver will handle each txn orderly,
they will acquire locks in 'PREPARE' phase in turn.

2. leader send replicated msg to follower, sending them together or
independent. T1, T2, T3 will be ordered to execute by follower and send
them back by together or independent in turn. raft_pool_token_ guarantee
T1, T2 T3 be ordered to apply.

3. T1, T2, T3 are putted to the queue of apply_pool_, order or concurrented
to be executed, releasing locks.

-----------

Contrast to origin, the keys locks are abandon, this can guarantee kudu
throughput more smoothly. What do you think? Hope to get your advice, kudu
users.

Thanks.
xiaokai.


---------- Forwarded message ---------
> From: Mike Percy <mp...@apache.org>
> Date: 2018年9月19日周三 上午8:29
> Subject: Re: Locks are acquired to cost much time in transactions
> To: <us...@kudu.apache.org>
>
>
> Why do you think you are spending a lot of time contending on row locks?
>
> Have you tried configuring your clients to send smaller batches? This may
> decrease throughput on a per-client basis but will likely improve latency
> and reduce the likelihood of row lock contention.
>
> If you are really spending most of your time contending on row locks then
> you will likely run into more fundamental performance issues trying to
> scale your writes, since Kudu's MVCC implementation effectively stores a
> linked list of updates to a given cell until compaction occurs. See
> https://github.com/apache/kudu/blob/master/docs/design-docs/tablet.md#historical-mvcc-in-diskrowsets
> for more information about the on-disk design.
>
> If you accumulate too many uncompacted mutations against a given row,
> reading the latest value for that row at scan time will be slow because it
> has to do a lot of work at read time.
>
> Mike
>
> On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com>
> wrote:
>
>> Moved here from JIRA.
>>
>> Hi guys, I met a problem about the keys locks that almost impacts the
>> service normal writing.
>>
>>
>> As we all know, a transaction which get all row_key locks will go on
>> next step in kudu. Everything looks good, if keys are not concurrent
>> updated. But when keys are updated by more than one client at the same time
>> , locks are acquired to wait much time. The cases are often in my
>> product environment. Does anybody meet the problem? Has any good ideal for
>> this?
>>
>>
>> In my way, I want to try to abandon keys locks, instead using
>> *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial
>> and ordered. Dose this work?
>>
>>
>> Hope to get your advice. Thanks.
>>
>>
>> -----
>> Regards,
>> Xiaokai
>>
>

Re: Locks are acquired to cost much time in transactions

Posted by Mike Percy <mp...@apache.org>.
Why do you think you are spending a lot of time contending on row locks?

Have you tried configuring your clients to send smaller batches? This may
decrease throughput on a per-client basis but will likely improve latency
and reduce the likelihood of row lock contention.

If you are really spending most of your time contending on row locks then
you will likely run into more fundamental performance issues trying to
scale your writes, since Kudu's MVCC implementation effectively stores a
linked list of updates to a given cell until compaction occurs. See
https://github.com/apache/kudu/blob/master/docs/design-docs/tablet.md#historical-mvcc-in-diskrowsets
for more information about the on-disk design.

If you accumulate too many uncompacted mutations against a given row,
reading the latest value for that row at scan time will be slow because it
has to do a lot of work at read time.

Mike

On Tue, Sep 18, 2018 at 8:48 AM Xiaokai Wang <xi...@live.com> wrote:

> Moved here from JIRA.
>
> Hi guys, I met a problem about the keys locks that almost impacts the
> service normal writing.
>
>
> As we all know, a transaction which get all row_key locks will go on next
> step in kudu. Everything looks good, if keys are not concurrent updated.
> But when keys are updated by more than one client at the same time, locks
> are acquired to wait much time. The cases are often in my product
> environment. Does anybody meet the problem? Has any good ideal for this?
>
>
> In my way, I want to try to abandon keys locks, instead using
> *_pool_token_ 'SERIAL' mode which keeping the key of transaction is serial
> and ordered. Dose this work?
>
>
> Hope to get your advice. Thanks.
>
>
> -----
> Regards,
> Xiaokai
>