You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by "Jonathan Ellis (JIRA)" <ji...@apache.org> on 2014/09/17 00:31:34 UTC

[jira] [Comment Edited] (CASSANDRA-7937) Apply backpressure gently when overloaded with writes

    [ https://issues.apache.org/jira/browse/CASSANDRA-7937?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14136381#comment-14136381 ] 

Jonathan Ellis edited comment on CASSANDRA-7937 at 9/16/14 10:30 PM:
---------------------------------------------------------------------

How do you backpressure in a distributed system?

Suppose a cluster has 5 nodes, A..E.  Node B is overloaded.  Client is talking to node A.  We then have some conflicting goals:

# Backpressuring from B to A does not help, it just pushes the problem into A's messagingservice
# So, we really want A to make the client slow down
# But, we do not want to restrict requests to CDE simply because B is overloaded.  Backpressure is too blunt a tool to restrict requests to B alone

I conclude that load shedding (dropping requests) is a better solution than backpressure for us.  


was (Author: jbellis):
How do you backpressure in a distributed system?

Suppose a cluster has 5 nodes, A..E.  Node B is overloaded.  Client is talking to node A.  We do not want to restrict requests to CDE simply because B is overloaded.

I conclude that load shedding (dropping requests) is a better solution than backpressure for us.

> Apply backpressure gently when overloaded with writes
> -----------------------------------------------------
>
>                 Key: CASSANDRA-7937
>                 URL: https://issues.apache.org/jira/browse/CASSANDRA-7937
>             Project: Cassandra
>          Issue Type: Bug
>          Components: Core
>         Environment: Cassandra 2.0
>            Reporter: Piotr Kołaczkowski
>              Labels: performance
>
> When writing huge amounts of data into C* cluster from analytic tools like Hadoop or Apache Spark, we can see that often C* can't keep up with the load. This is because analytic tools typically write data "as fast as they can" in parallel, from many nodes and they are not artificially rate-limited, so C* is the bottleneck here. Also, increasing the number of nodes doesn't really help, because in a collocated setup this also increases number of Hadoop/Spark nodes (writers) and although possible write performance is higher, the problem still remains.
> We observe the following behavior:
> 1. data is ingested at an extreme fast pace into memtables and flush queue fills up
> 2. the available memory limit for memtables is reached and writes are no longer accepted
> 3. the application gets hit by "write timeout", and retries repeatedly, in vain 
> 4. after several failed attempts to write, the job gets aborted 
> Desired behaviour:
> 1. data is ingested at an extreme fast pace into memtables and flush queue fills up
> 2. after exceeding some memtable "fill threshold", C* applies adaptive rate limiting to writes - the more the buffers are filled-up, the less writes/s are accepted, however writes still occur within the write timeout.
> 3. thanks to slowed down data ingestion, now flush can finish before all the memory gets used
> Of course the details how rate limiting could be done are up for a discussion.
> It may be also worth considering putting such logic into the driver, not C* core, but then C* needs to expose at least the following information to the driver, so we could calculate the desired maximum data rate:
> 1. current amount of memory available for writes before they would completely block
> 2. total amount of data queued to be flushed and flush progress (amount of data to flush remaining for the memtable currently being flushed)
> 3. average flush write speed



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)