You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@cassandra.apache.org by Yuji Ito <yu...@imagine-orb.com> on 2017/05/13 12:21:43 UTC

Proposal - GroupCommitLogService

Hi dev,

I propose a new CommitLogService, GroupCommitLogService, to improve the
throughput when lots of requests are received.
It improved the throughput by maximum 94%.
I'd like to discuss about this CommitLogService.

Currently, we can select either 2 CommitLog services; Periodic and Batch.
In Periodic, we might lose some commit log which hasn't written to the disk.
In Batch, we can write commit log to the disk every time. The size of
commit log to write is too small (< 4KB). When high concurrency, these
writes are gathered and persisted to the disk at once. But, when
insufficient concurrency, many small writes are issued and the performance
decreases due to the latency of the disk. Even if you use SSD, processes of
many IO commands decrease the performance.

GroupCommitLogService writes some commitlog to the disk at once.
The patch adds GroupCommitLogService (It is enabled by setting
`commitlog_sync` and `commitlog_sync_group_window_in_ms` in cassandra.yaml).
The difference from Batch is just only waiting for the semaphore.
By waiting for the semaphore, some writes for commit logs are executed at
the same time.
In GroupCommitLogService, the latency becomes worse if the there is no
concurrency.

I measured the performance with my microbench (MicroRequestThread.java) by
increasing the number of threads.The cluster has 3 nodes (Replication
factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
The result is as below. The GroupCommitLogService with 10ms window improved
update with Paxos by 94% and improved select with Paxos by 76%.

==== SELECT / sec ====
# of threads Batch 2ms Group 10ms
1 192 103
2 163 212
4 264 416
8 454 800
16 744 1311
32 1151 1481
64 1767 1844
128 2949 3011
256 4723 5000

==== UPDATE / sec ====
# of threads Batch 2ms Group 10ms
1 45 26
2 39 51
4 58 102
8 102 198
16 167 213
32 289 295
64 544 548
128 1046 1058
256 2020 2061


Thanks,
Yuji

Re: Proposal - GroupCommitLogService

Posted by Yuji Ito <yu...@imagine-orb.com>.
Thanks Jeremiah,

I've opened a ticket on JIRA.

https://issues.apache.org/jira/browse/CASSANDRA-13530

Best,
Yuji


On Sat, May 13, 2017 at 9:38 PM, J. D. Jordan <je...@gmail.com>
wrote:

> Sounds interesting. You should open a JIRA and attach your code for
> discussion of it.
>
> https://issues.apache.org/jira/browse/CASSANDRA/
> <https://issues.apache.org/jira/browse/CASSANDRA/?selectedTab=com.atlassian.jira.jira-projects-plugin:summary-panel>
>
> -Jeremiah
>
> On May 13, 2017, at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>
> Hi dev,
>
> I propose a new CommitLogService, GroupCommitLogService, to improve the
> throughput when lots of requests are received.
> It improved the throughput by maximum 94%.
> I'd like to discuss about this CommitLogService.
>
> Currently, we can select either 2 CommitLog services; Periodic and Batch.
> In Periodic, we might lose some commit log which hasn't written to the
> disk.
> In Batch, we can write commit log to the disk every time. The size of
> commit log to write is too small (< 4KB). When high concurrency, these
> writes are gathered and persisted to the disk at once. But, when
> insufficient concurrency, many small writes are issued and the performance
> decreases due to the latency of the disk. Even if you use SSD, processes of
> many IO commands decrease the performance.
>
> GroupCommitLogService writes some commitlog to the disk at once.
> The patch adds GroupCommitLogService (It is enabled by setting
> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
> cassandra.yaml).
> The difference from Batch is just only waiting for the semaphore.
> By waiting for the semaphore, some writes for commit logs are executed at
> the same time.
> In GroupCommitLogService, the latency becomes worse if the there is no
> concurrency.
>
> I measured the performance with my microbench (MicroRequestThread.java) by
> increasing the number of threads.The cluster has 3 nodes (Replication
> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
> The result is as below. The GroupCommitLogService with 10ms window
> improved update with Paxos by 94% and improved select with Paxos by 76%.
>
> ==== SELECT / sec ====
> # of threads Batch 2ms Group 10ms
> 1 192 103
> 2 163 212
> 4 264 416
> 8 454 800
> 16 744 1311
> 32 1151 1481
> 64 1767 1844
> 128 2949 3011
> 256 4723 5000
>
> ==== UPDATE / sec ====
> # of threads Batch 2ms Group 10ms
> 1 45 26
> 2 39 51
> 4 58 102
> 8 102 198
> 16 167 213
> 32 289 295
> 64 544 548
> 128 1046 1058
> 256 2020 2061
>
>
> Thanks,
> Yuji
>
>
> ---------------------------------------------------------------------
> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
> For additional commands, e-mail: dev-help@cassandra.apache.org
>
>

Re: Proposal - GroupCommitLogService

Posted by "J. D. Jordan" <je...@gmail.com>.
Sounds interesting. You should open a JIRA and attach your code for discussion of it.

https://issues.apache.org/jira/browse/CASSANDRA/

-Jeremiah

> On May 13, 2017, at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
> 
> Hi dev,
> 
> I propose a new CommitLogService, GroupCommitLogService, to improve the throughput when lots of requests are received.
> It improved the throughput by maximum 94%.
> I'd like to discuss about this CommitLogService.
> 
> Currently, we can select either 2 CommitLog services; Periodic and Batch.
> In Periodic, we might lose some commit log which hasn't written to the disk.
> In Batch, we can write commit log to the disk every time. The size of commit log to write is too small (< 4KB). When high concurrency, these writes are gathered and persisted to the disk at once. But, when insufficient concurrency, many small writes are issued and the performance decreases due to the latency of the disk. Even if you use SSD, processes of many IO commands decrease the performance.
> 
> GroupCommitLogService writes some commitlog to the disk at once.
> The patch adds GroupCommitLogService (It is enabled by setting `commitlog_sync` and `commitlog_sync_group_window_in_ms` in cassandra.yaml).
> The difference from Batch is just only waiting for the semaphore.
> By waiting for the semaphore, some writes for commit logs are executed at the same time.
> In GroupCommitLogService, the latency becomes worse if the there is no concurrency.
> 
> I measured the performance with my microbench (MicroRequestThread.java) by increasing the number of threads.The cluster has 3 nodes (Replication factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
> The result is as below. The GroupCommitLogService with 10ms window improved update with Paxos by 94% and improved select with Paxos by 76%.
> 
> ==== SELECT / sec ====
> # of threads	Batch 2ms	Group 10ms
> 1	192	103
> 2	163	212
> 4	264	416
> 8	454	800
> 16	744	1311
> 32	1151	1481
> 64	1767	1844
> 128	2949	3011
> 256	4723	5000
> 
> ==== UPDATE / sec ====
> # of threads	Batch 2ms	Group 10ms
> 1	45	26
> 2	39	51
> 4	58	102
> 8	102	198
> 16	167	213
> 32	289	295
> 64	544	548
> 128	1046	1058
> 256	2020	2061
> 
> 
> Thanks,
> Yuji
> 
> 
> ---------------------------------------------------------------------
> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
> For additional commands, e-mail: dev-help@cassandra.apache.org

Re: Proposal - GroupCommitLogService

Posted by Hiroyuki Yamada <mo...@gmail.com>.
Hi all,

> Yuji
Great work !
That's what I expected for batch.
(I was the one who wrote the jira below.
<https://issues.apache.org/jira/browse/CASSANDRA-12864>)

I think it might be better for us to have your proposed group-commit
feature as "batch"
instead of having another commit-log-service.

Thanks,
Hiro




On Sun, May 14, 2017 at 10:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
> That's exactly the motivation of this proposal.
>
> Batch can group only writes which are not persisted (not kept waiting) at
> that time.
> In Batch, a write of commitlog isn't kept waiting because the thread lock
> (the semaphore in 2.2 and 3.0) for sync is released immediately.
>
> So, the window size means 'the maximum length of time that queries may be
> batched together for, not the minimum'.
> The Batch window size doesn't almost affect the performance.
> https://issues.apache.org/jira/browse/CASSANDRA-12864
>
> I tested the throughput of SELECT with Batch window 10ms.
> The result was the same as Batch window 2ms as expected.
>
> ==== SELECT / sec ====
> # of threads batch 2ms batch 10ms
> 1 192 192
> 2 163 169
> 4 264 263
> 8 454 454
> 16 744 744
> 32 1151 1155
> 64 1767 1772
> 128 2949 2962
> 256 4723 4785
>
>
> Yuji
>
>
> On Sun, May 14, 2017 at 2:51 AM, Jonathan Ellis <jb...@gmail.com> wrote:
>
>> Does that mean that Batch is not working as designed?  If there are other
>> pending writes, Batch should also group them together.  (Did you test with
>> giving Batch the same window size as Group?)
>>
>> On Sat, May 13, 2017 at 10:08 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>>
>>> Batch outperforms when there is no concurrency.
>>> Because GroupCommit should wait the window time, the throughput and the
>>> latency are worse in a single request.
>>> GroupCommit can gather the commitlog writes which are requested in the
>>> window time.
>>> Actually, the throughput of a single thread was bounded by the window
>>> time.
>>>
>>> Yuji
>>>
>>> On Sat, May 13, 2017 at 11:49 PM, Jonathan Ellis <jb...@gmail.com>
>>> wrote:
>>>
>>>> Can we replace Batch entirely with this, or are there situations where
>>>> Batch would outperform (in latency, for instance)?
>>>>
>>>> On Sat, May 13, 2017 at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>>>>
>>>>> Hi dev,
>>>>>
>>>>> I propose a new CommitLogService, GroupCommitLogService, to improve the
>>>>> throughput when lots of requests are received.
>>>>> It improved the throughput by maximum 94%.
>>>>> I'd like to discuss about this CommitLogService.
>>>>>
>>>>> Currently, we can select either 2 CommitLog services; Periodic and
>>>>> Batch.
>>>>> In Periodic, we might lose some commit log which hasn't written to the
>>>>> disk.
>>>>> In Batch, we can write commit log to the disk every time. The size of
>>>>> commit log to write is too small (< 4KB). When high concurrency, these
>>>>> writes are gathered and persisted to the disk at once. But, when
>>>>> insufficient concurrency, many small writes are issued and the performance
>>>>> decreases due to the latency of the disk. Even if you use SSD, processes of
>>>>> many IO commands decrease the performance.
>>>>>
>>>>> GroupCommitLogService writes some commitlog to the disk at once.
>>>>> The patch adds GroupCommitLogService (It is enabled by setting
>>>>> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
>>>>> cassandra.yaml).
>>>>> The difference from Batch is just only waiting for the semaphore.
>>>>> By waiting for the semaphore, some writes for commit logs are executed
>>>>> at the same time.
>>>>> In GroupCommitLogService, the latency becomes worse if the there is no
>>>>> concurrency.
>>>>>
>>>>> I measured the performance with my microbench (MicroRequestThread.java)
>>>>> by increasing the number of threads.The cluster has 3 nodes (Replication
>>>>> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
>>>>> The result is as below. The GroupCommitLogService with 10ms window
>>>>> improved update with Paxos by 94% and improved select with Paxos by 76%.
>>>>>
>>>>> ==== SELECT / sec ====
>>>>> # of threads Batch 2ms Group 10ms
>>>>> 1 192 103
>>>>> 2 163 212
>>>>> 4 264 416
>>>>> 8 454 800
>>>>> 16 744 1311
>>>>> 32 1151 1481
>>>>> 64 1767 1844
>>>>> 128 2949 3011
>>>>> 256 4723 5000
>>>>>
>>>>> ==== UPDATE / sec ====
>>>>> # of threads Batch 2ms Group 10ms
>>>>> 1 45 26
>>>>> 2 39 51
>>>>> 4 58 102
>>>>> 8 102 198
>>>>> 16 167 213
>>>>> 32 289 295
>>>>> 64 544 548
>>>>> 128 1046 1058
>>>>> 256 2020 2061
>>>>>
>>>>>
>>>>> Thanks,
>>>>> Yuji
>>>>>
>>>>>
>>>>>
>>>>> ---------------------------------------------------------------------
>>>>> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
>>>>> For additional commands, e-mail: dev-help@cassandra.apache.org
>>>>>
>>>>
>>>>
>>>>
>>>> --
>>>> Jonathan Ellis
>>>> co-founder, http://www.datastax.com
>>>> @spyced
>>>>
>>>
>>>
>>
>>
>> --
>> Jonathan Ellis
>> co-founder, http://www.datastax.com
>> @spyced
>>

---------------------------------------------------------------------
To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
For additional commands, e-mail: dev-help@cassandra.apache.org


Re: Proposal - GroupCommitLogService

Posted by Yuji Ito <yu...@imagine-orb.com>.
That's exactly the motivation of this proposal.

Batch can group only writes which are not persisted (not kept waiting) at
that time.
In Batch, a write of commitlog isn't kept waiting because the thread lock
(the semaphore in 2.2 and 3.0) for sync is released immediately.

So, the window size means 'the maximum length of time that queries may be
batched together for, not the minimum'.
The Batch window size doesn't almost affect the performance.
https://issues.apache.org/jira/browse/CASSANDRA-12864

I tested the throughput of SELECT with Batch window 10ms.
The result was the same as Batch window 2ms as expected.

==== SELECT / sec ====
# of threads batch 2ms batch 10ms
1 192 192
2 163 169
4 264 263
8 454 454
16 744 744
32 1151 1155
64 1767 1772
128 2949 2962
256 4723 4785


Yuji


On Sun, May 14, 2017 at 2:51 AM, Jonathan Ellis <jb...@gmail.com> wrote:

> Does that mean that Batch is not working as designed?  If there are other
> pending writes, Batch should also group them together.  (Did you test with
> giving Batch the same window size as Group?)
>
> On Sat, May 13, 2017 at 10:08 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>
>> Batch outperforms when there is no concurrency.
>> Because GroupCommit should wait the window time, the throughput and the
>> latency are worse in a single request.
>> GroupCommit can gather the commitlog writes which are requested in the
>> window time.
>> Actually, the throughput of a single thread was bounded by the window
>> time.
>>
>> Yuji
>>
>> On Sat, May 13, 2017 at 11:49 PM, Jonathan Ellis <jb...@gmail.com>
>> wrote:
>>
>>> Can we replace Batch entirely with this, or are there situations where
>>> Batch would outperform (in latency, for instance)?
>>>
>>> On Sat, May 13, 2017 at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>>>
>>>> Hi dev,
>>>>
>>>> I propose a new CommitLogService, GroupCommitLogService, to improve the
>>>> throughput when lots of requests are received.
>>>> It improved the throughput by maximum 94%.
>>>> I'd like to discuss about this CommitLogService.
>>>>
>>>> Currently, we can select either 2 CommitLog services; Periodic and
>>>> Batch.
>>>> In Periodic, we might lose some commit log which hasn't written to the
>>>> disk.
>>>> In Batch, we can write commit log to the disk every time. The size of
>>>> commit log to write is too small (< 4KB). When high concurrency, these
>>>> writes are gathered and persisted to the disk at once. But, when
>>>> insufficient concurrency, many small writes are issued and the performance
>>>> decreases due to the latency of the disk. Even if you use SSD, processes of
>>>> many IO commands decrease the performance.
>>>>
>>>> GroupCommitLogService writes some commitlog to the disk at once.
>>>> The patch adds GroupCommitLogService (It is enabled by setting
>>>> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
>>>> cassandra.yaml).
>>>> The difference from Batch is just only waiting for the semaphore.
>>>> By waiting for the semaphore, some writes for commit logs are executed
>>>> at the same time.
>>>> In GroupCommitLogService, the latency becomes worse if the there is no
>>>> concurrency.
>>>>
>>>> I measured the performance with my microbench (MicroRequestThread.java)
>>>> by increasing the number of threads.The cluster has 3 nodes (Replication
>>>> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
>>>> The result is as below. The GroupCommitLogService with 10ms window
>>>> improved update with Paxos by 94% and improved select with Paxos by 76%.
>>>>
>>>> ==== SELECT / sec ====
>>>> # of threads Batch 2ms Group 10ms
>>>> 1 192 103
>>>> 2 163 212
>>>> 4 264 416
>>>> 8 454 800
>>>> 16 744 1311
>>>> 32 1151 1481
>>>> 64 1767 1844
>>>> 128 2949 3011
>>>> 256 4723 5000
>>>>
>>>> ==== UPDATE / sec ====
>>>> # of threads Batch 2ms Group 10ms
>>>> 1 45 26
>>>> 2 39 51
>>>> 4 58 102
>>>> 8 102 198
>>>> 16 167 213
>>>> 32 289 295
>>>> 64 544 548
>>>> 128 1046 1058
>>>> 256 2020 2061
>>>>
>>>>
>>>> Thanks,
>>>> Yuji
>>>>
>>>>
>>>>
>>>> ---------------------------------------------------------------------
>>>> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
>>>> For additional commands, e-mail: dev-help@cassandra.apache.org
>>>>
>>>
>>>
>>>
>>> --
>>> Jonathan Ellis
>>> co-founder, http://www.datastax.com
>>> @spyced
>>>
>>
>>
>
>
> --
> Jonathan Ellis
> co-founder, http://www.datastax.com
> @spyced
>

Re: Proposal - GroupCommitLogService

Posted by Jonathan Ellis <jb...@gmail.com>.
Does that mean that Batch is not working as designed?  If there are other
pending writes, Batch should also group them together.  (Did you test with
giving Batch the same window size as Group?)

On Sat, May 13, 2017 at 10:08 AM, Yuji Ito <yu...@imagine-orb.com> wrote:

> Batch outperforms when there is no concurrency.
> Because GroupCommit should wait the window time, the throughput and the
> latency are worse in a single request.
> GroupCommit can gather the commitlog writes which are requested in the
> window time.
> Actually, the throughput of a single thread was bounded by the window time.
>
> Yuji
>
> On Sat, May 13, 2017 at 11:49 PM, Jonathan Ellis <jb...@gmail.com>
> wrote:
>
>> Can we replace Batch entirely with this, or are there situations where
>> Batch would outperform (in latency, for instance)?
>>
>> On Sat, May 13, 2017 at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>>
>>> Hi dev,
>>>
>>> I propose a new CommitLogService, GroupCommitLogService, to improve the
>>> throughput when lots of requests are received.
>>> It improved the throughput by maximum 94%.
>>> I'd like to discuss about this CommitLogService.
>>>
>>> Currently, we can select either 2 CommitLog services; Periodic and Batch.
>>> In Periodic, we might lose some commit log which hasn't written to the
>>> disk.
>>> In Batch, we can write commit log to the disk every time. The size of
>>> commit log to write is too small (< 4KB). When high concurrency, these
>>> writes are gathered and persisted to the disk at once. But, when
>>> insufficient concurrency, many small writes are issued and the performance
>>> decreases due to the latency of the disk. Even if you use SSD, processes of
>>> many IO commands decrease the performance.
>>>
>>> GroupCommitLogService writes some commitlog to the disk at once.
>>> The patch adds GroupCommitLogService (It is enabled by setting
>>> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
>>> cassandra.yaml).
>>> The difference from Batch is just only waiting for the semaphore.
>>> By waiting for the semaphore, some writes for commit logs are executed
>>> at the same time.
>>> In GroupCommitLogService, the latency becomes worse if the there is no
>>> concurrency.
>>>
>>> I measured the performance with my microbench (MicroRequestThread.java)
>>> by increasing the number of threads.The cluster has 3 nodes (Replication
>>> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
>>> The result is as below. The GroupCommitLogService with 10ms window
>>> improved update with Paxos by 94% and improved select with Paxos by 76%.
>>>
>>> ==== SELECT / sec ====
>>> # of threads Batch 2ms Group 10ms
>>> 1 192 103
>>> 2 163 212
>>> 4 264 416
>>> 8 454 800
>>> 16 744 1311
>>> 32 1151 1481
>>> 64 1767 1844
>>> 128 2949 3011
>>> 256 4723 5000
>>>
>>> ==== UPDATE / sec ====
>>> # of threads Batch 2ms Group 10ms
>>> 1 45 26
>>> 2 39 51
>>> 4 58 102
>>> 8 102 198
>>> 16 167 213
>>> 32 289 295
>>> 64 544 548
>>> 128 1046 1058
>>> 256 2020 2061
>>>
>>>
>>> Thanks,
>>> Yuji
>>>
>>>
>>>
>>> ---------------------------------------------------------------------
>>> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
>>> For additional commands, e-mail: dev-help@cassandra.apache.org
>>>
>>
>>
>>
>> --
>> Jonathan Ellis
>> co-founder, http://www.datastax.com
>> @spyced
>>
>
>


-- 
Jonathan Ellis
co-founder, http://www.datastax.com
@spyced

Re: Proposal - GroupCommitLogService

Posted by Yuji Ito <yu...@imagine-orb.com>.
Batch outperforms when there is no concurrency.
Because GroupCommit should wait the window time, the throughput and the
latency are worse in a single request.
GroupCommit can gather the commitlog writes which are requested in the
window time.
Actually, the throughput of a single thread was bounded by the window time.

Yuji

On Sat, May 13, 2017 at 11:49 PM, Jonathan Ellis <jb...@gmail.com> wrote:

> Can we replace Batch entirely with this, or are there situations where
> Batch would outperform (in latency, for instance)?
>
> On Sat, May 13, 2017 at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:
>
>> Hi dev,
>>
>> I propose a new CommitLogService, GroupCommitLogService, to improve the
>> throughput when lots of requests are received.
>> It improved the throughput by maximum 94%.
>> I'd like to discuss about this CommitLogService.
>>
>> Currently, we can select either 2 CommitLog services; Periodic and Batch.
>> In Periodic, we might lose some commit log which hasn't written to the
>> disk.
>> In Batch, we can write commit log to the disk every time. The size of
>> commit log to write is too small (< 4KB). When high concurrency, these
>> writes are gathered and persisted to the disk at once. But, when
>> insufficient concurrency, many small writes are issued and the performance
>> decreases due to the latency of the disk. Even if you use SSD, processes of
>> many IO commands decrease the performance.
>>
>> GroupCommitLogService writes some commitlog to the disk at once.
>> The patch adds GroupCommitLogService (It is enabled by setting
>> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
>> cassandra.yaml).
>> The difference from Batch is just only waiting for the semaphore.
>> By waiting for the semaphore, some writes for commit logs are executed at
>> the same time.
>> In GroupCommitLogService, the latency becomes worse if the there is no
>> concurrency.
>>
>> I measured the performance with my microbench (MicroRequestThread.java)
>> by increasing the number of threads.The cluster has 3 nodes (Replication
>> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
>> The result is as below. The GroupCommitLogService with 10ms window
>> improved update with Paxos by 94% and improved select with Paxos by 76%.
>>
>> ==== SELECT / sec ====
>> # of threads Batch 2ms Group 10ms
>> 1 192 103
>> 2 163 212
>> 4 264 416
>> 8 454 800
>> 16 744 1311
>> 32 1151 1481
>> 64 1767 1844
>> 128 2949 3011
>> 256 4723 5000
>>
>> ==== UPDATE / sec ====
>> # of threads Batch 2ms Group 10ms
>> 1 45 26
>> 2 39 51
>> 4 58 102
>> 8 102 198
>> 16 167 213
>> 32 289 295
>> 64 544 548
>> 128 1046 1058
>> 256 2020 2061
>>
>>
>> Thanks,
>> Yuji
>>
>>
>>
>> ---------------------------------------------------------------------
>> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
>> For additional commands, e-mail: dev-help@cassandra.apache.org
>>
>
>
>
> --
> Jonathan Ellis
> co-founder, http://www.datastax.com
> @spyced
>

Re: Proposal - GroupCommitLogService

Posted by Jonathan Ellis <jb...@gmail.com>.
Can we replace Batch entirely with this, or are there situations where
Batch would outperform (in latency, for instance)?

On Sat, May 13, 2017 at 7:21 AM, Yuji Ito <yu...@imagine-orb.com> wrote:

> Hi dev,
>
> I propose a new CommitLogService, GroupCommitLogService, to improve the
> throughput when lots of requests are received.
> It improved the throughput by maximum 94%.
> I'd like to discuss about this CommitLogService.
>
> Currently, we can select either 2 CommitLog services; Periodic and Batch.
> In Periodic, we might lose some commit log which hasn't written to the
> disk.
> In Batch, we can write commit log to the disk every time. The size of
> commit log to write is too small (< 4KB). When high concurrency, these
> writes are gathered and persisted to the disk at once. But, when
> insufficient concurrency, many small writes are issued and the performance
> decreases due to the latency of the disk. Even if you use SSD, processes of
> many IO commands decrease the performance.
>
> GroupCommitLogService writes some commitlog to the disk at once.
> The patch adds GroupCommitLogService (It is enabled by setting
> `commitlog_sync` and `commitlog_sync_group_window_in_ms` in
> cassandra.yaml).
> The difference from Batch is just only waiting for the semaphore.
> By waiting for the semaphore, some writes for commit logs are executed at
> the same time.
> In GroupCommitLogService, the latency becomes worse if the there is no
> concurrency.
>
> I measured the performance with my microbench (MicroRequestThread.java) by
> increasing the number of threads.The cluster has 3 nodes (Replication
> factor: 3). Each nodes is AWS EC2 m4.large instance + 200IOPS io1 volume.
> The result is as below. The GroupCommitLogService with 10ms window
> improved update with Paxos by 94% and improved select with Paxos by 76%.
>
> ==== SELECT / sec ====
> # of threads Batch 2ms Group 10ms
> 1 192 103
> 2 163 212
> 4 264 416
> 8 454 800
> 16 744 1311
> 32 1151 1481
> 64 1767 1844
> 128 2949 3011
> 256 4723 5000
>
> ==== UPDATE / sec ====
> # of threads Batch 2ms Group 10ms
> 1 45 26
> 2 39 51
> 4 58 102
> 8 102 198
> 16 167 213
> 32 289 295
> 64 544 548
> 128 1046 1058
> 256 2020 2061
>
>
> Thanks,
> Yuji
>
>
>
> ---------------------------------------------------------------------
> To unsubscribe, e-mail: dev-unsubscribe@cassandra.apache.org
> For additional commands, e-mail: dev-help@cassandra.apache.org
>



-- 
Jonathan Ellis
co-founder, http://www.datastax.com
@spyced