You are viewing a plain text version of this content. The canonical link for it is here.
Posted to user@cassandra.apache.org by Martin Krasser <kr...@googlemail.com> on 2015/05/18 12:32:07 UTC

Batch isolation within a single partition

Hello,

I have an application that inserts multiple rows within a single 
partition (= all rows share the same partition key) using a BATCH 
statement. Is it possible that other clients can partially read that 
batch or is the batch application isolated i.e. other clients can only 
read all rows of that batch or none of them?

I understand that a BATCH update to multiple partitions is not isolated 
but I'm not sure if this is also the case for a single partition:

- The article Atomic batches in Cassandra 1.2 
<http://www.datastax.com/dev/blog/atomic-batches-in-cassandra-1-2> says 
that /"... we mean atomic in the database sense that if any part of the 
batch succeeds, all of it will. No other guarantees are implied; in 
particular, there is no isolation"/.

- On the other hand, the CQL BATCH 
<https://cassandra.apache.org/doc/cql3/CQL.html#batchStmt> docs at 
cassandra.apache.org mention that /"/ /... the [batch] operations are 
still only isolated within a single partition"/ which is a clear 
statement but doesn't it contradict the previous and the next one?

- The CQL BATCH 
<http://docs.datastax.com/en/cql/3.1/cql/cql_reference/batch_r.html> 
docs at docs.datastax.com mention that /"... there is no batch 
isolation. Clients are able to read the first updated rows from the 
batch, while other rows are still being updated on the server. However, 
transactional row updates within a partition key are isolated: clients 
cannot read a partial update"/. Also, what does /"transactional row 
updates"/ mean in this context? A lightweight transaction? Something else?

Thanks for any help,
Martin


Re: Batch isolation within a single partition

Posted by Martin Krasser <kr...@googlemail.com>.

On 19.05.15 10:04, Sylvain Lebresne wrote:
> On Tue, May 19, 2015 at 9:42 AM, DuyHai Doan <doanduyhai@gmail.com 
> <ma...@gmail.com>> wrote:
>
>     If RF > 1, the consistency level at QUORUM cannot guarantee strict
>     isolation (for normal mutation or batch). If you look at this
>     slide:
>     http://www.slideshare.net/doanduyhai/cassandra-introduction-apache-con-2014-budapest/25,
>     you can see that the mutation is sent by the coordinator, in
>     parallel, to all replicas.
>
>      Now it is very possible that due to network latency, the mutation
>     is applied on the first replica and is applied with "some delay"
>     (which can be at the order of microseconds) on other replicas.
>
>      Theoretically, one client can read updated value on first replica
>     and old value on the other replicas, even at QUORUM.
>
>
> Unfortunately different people will tend to have different definitions 
> of isolation and I don't seem to have the same definition than you but 
> still, I don't understand what you're talking about. Of course 
> replicas might not get a mutation at the same time, and yes, a read at 
> QUORUM may thus not see the most up to date value from all replicas.

If I understand correctly, this can only happen if a QUORUM read started 
*before* the QUORUM write completed. If, on the other hand, a QUORUM 
read follows a *completed* QUORUM write, shouldn't the read always 
return the most recent value?

For example, with RF = 3 and QUORUM write + read, we have nodes_written 
+ nodes_read > RF (with nodes_written = nodes_read = 2) which guarantees 
consistency, or am I missing something?

> But the coordinator resolves all responses together and return only 
> the most recent one, so that doesn't matter to the client and I don't 
> see how that has anything to do with isolation from the client 
> perspective.

+1

>
> My response to the original question is that if by isolation you mean 
> "can a reader observe a write only partially applied", then for single 
> partition writes, Cassandra do offer isolation.

Yes, this is exactly what I mean (and what I need for batch writes to a 
single partition).

> One caveat however is that if 2 writes conflits, they are resolved 
> using their timestamp and if the timestamp are the same, resolution is 
> based on values, which is not necessarily intuitive and may make it 
> sound like the writes where not applied in isolation (even though 
> technically they are), see 
> https://issues.apache.org/jira/browse/CASSANDRA-6123 for details on 
> that later problem. I'll note that my definition of isolation does not 
> mean you can't read stale data, and you can indeed if you use weak 
> consistency levels.

I completely share your view/definition of isolation - it's not about 
staleness, it's only about that a reader cannot observe partial writes.

Regarding staleness/consistency, if I want to read the most recent 
batch-write a QUORUM read must follow the completed QUORUM (batch) 
write, right?

Thanks for your clarifications,
Martin

>
> If you mean something else by isolation, then I think agreeing first 
> on the definition would be wise.
>
> --
> Sylvain

-- 
Martin Krasser

blog:    http://krasserm.github.io
code:    http://github.com/krasserm
twitter: http://twitter.com/mrt1nz


Re: Batch isolation within a single partition

Posted by Sylvain Lebresne <sy...@datastax.com>.
On Tue, May 19, 2015 at 9:42 AM, DuyHai Doan <do...@gmail.com> wrote:

> If RF > 1, the consistency level at QUORUM cannot guarantee strict
> isolation (for normal mutation or batch). If you look at this slide:
> http://www.slideshare.net/doanduyhai/cassandra-introduction-apache-con-2014-budapest/25,
> you can see that the mutation is sent by the coordinator, in parallel, to
> all replicas.
>
>  Now it is very possible that due to network latency, the mutation is
> applied on the first replica and is applied with "some delay" (which can be
> at the order of microseconds) on other replicas.
>
>  Theoretically, one client can read updated value on first replica and old
> value on the other replicas, even at QUORUM.
>

Unfortunately different people will tend to have different definitions of
isolation and I don't seem to have the same definition than you but still,
I don't understand what you're talking about. Of course replicas might not
get a mutation at the same time, and yes, a read at QUORUM may thus not see
the most up to date value from all replicas. But the coordinator resolves
all responses together and return only the most recent one, so that doesn't
matter to the client and I don't see how that has anything to do with
isolation from the client perspective.

My response to the original question is that if by isolation you mean "can
a reader observe a write only partially applied", then for single partition
writes, Cassandra do offer isolation. One caveat however is that if 2
writes conflits, they are resolved using their timestamp and if the
timestamp are the same, resolution is based on values, which is not
necessarily intuitive and may make it sound like the writes where not
applied in isolation (even though technically they are), see
https://issues.apache.org/jira/browse/CASSANDRA-6123 for details on that
later problem. I'll note that my definition of isolation does not mean you
can't read stale data, and you can indeed if you use weak consistency
levels.

If you mean something else by isolation, then I think agreeing first on the
definition would be wise.

--
Sylvain

Re: Batch isolation within a single partition

Posted by DuyHai Doan <do...@gmail.com>.
If RF > 1, the consistency level at QUORUM cannot guarantee strict
isolation (for normal mutation or batch). If you look at this slide:
http://www.slideshare.net/doanduyhai/cassandra-introduction-apache-con-2014-budapest/25,
you can see that the mutation is sent by the coordinator, in parallel, to
all replicas.

 Now it is very possible that due to network latency, the mutation is
applied on the first replica and is applied with "some delay" (which can be
at the order of microseconds) on other replicas.

 Theoretically, one client can read updated value on first replica and old
value on the other replicas, even at QUORUM.

 I think that reading at ALL may guarantee to have a "read isolation" in
practice but I'm not sure it can be considered "isolated" from the
theoretical definition of isolation.


On Tue, May 19, 2015 at 7:57 AM, Martin Krasser <kr...@googlemail.com>
wrote:

>  Hi DuyHai,
>
> thanks for your answer. What if I set RF > 1 and the consistency level for
> reads and writes to QUORUM? Would that isolate the single-partition batch
> update from reads? (I do not consider node failures here between the write
> and the read(s)).
>
>
> On 19.05.15 07:50, DuyHai Doan wrote:
>
> Hello Martin
>
>  If, and only if you have RF=1, single partition mutations (including
> batches) are isolated.
>
>  Otherwise, with RF>1, even a simple UPDATE is not isolated because one
> client can read the updated value on one replica and another client reads
> the old value on another replica
>
>
>
> On Mon, May 18, 2015 at 12:32 PM, Martin Krasser <kr...@googlemail.com>
> wrote:
>
>>  Hello,
>>
>> I have an application that inserts multiple rows within a single
>> partition (= all rows share the same partition key) using a BATCH
>> statement. Is it possible that other clients can partially read that batch
>> or is the batch application isolated i.e. other clients can only read all
>> rows of that batch or none of them?
>>
>> I understand that a BATCH update to multiple partitions is not isolated
>> but I'm not sure if this is also the case for a single partition:
>>
>> - The article Atomic batches in Cassandra 1.2
>> <http://www.datastax.com/dev/blog/atomic-batches-in-cassandra-1-2> says
>> that *"... we mean atomic in the database sense that if any part of the
>> batch succeeds, all of it will. No other guarantees are implied; in
>> particular, there is no isolation"*.
>>
>> - On the other hand, the CQL BATCH
>> <https://cassandra.apache.org/doc/cql3/CQL.html#batchStmt> docs at
>> cassandra.apache.org mention that *"* *... the [batch] operations are
>> still only isolated within a single partition"* which is a clear
>> statement but doesn't it contradict the previous and the next one?
>>
>> - The CQL BATCH
>> <http://docs.datastax.com/en/cql/3.1/cql/cql_reference/batch_r.html>
>> docs at docs.datastax.com mention that *"... there is no batch
>> isolation. Clients are able to read the first updated rows from the batch,
>> while other rows are still being updated on the server. However,
>> transactional row updates within a partition key are isolated: clients
>> cannot read a partial update"*. Also, what does *"transactional row
>> updates"* mean in this context? A lightweight transaction? Something
>> else?
>>
>> Thanks for any help,
>> Martin
>>
>>
>
> --
> Martin Krasser
>
> blog:    http://krasserm.github.io
> code:    http://github.com/krasserm
> twitter: http://twitter.com/mrt1nz
>
>

Re: Batch isolation within a single partition

Posted by Martin Krasser <kr...@googlemail.com>.
Hi DuyHai,

thanks for your answer. What if I set RF > 1 and the consistency level 
for reads and writes to QUORUM? Would that isolate the single-partition 
batch update from reads? (I do not consider node failures here between 
the write and the read(s)).

On 19.05.15 07:50, DuyHai Doan wrote:
> Hello Martin
>
> If, and only if you have RF=1, single partition mutations (including 
> batches) are isolated.
>
> Otherwise, with RF>1, even a simple UPDATE is not isolated because one 
> client can read the updated value on one replica and another client 
> reads the old value on another replica
>
>
>
> On Mon, May 18, 2015 at 12:32 PM, Martin Krasser 
> <krasserm@googlemail.com <ma...@googlemail.com>> wrote:
>
>     Hello,
>
>     I have an application that inserts multiple rows within a single
>     partition (= all rows share the same partition key) using a BATCH
>     statement. Is it possible that other clients can partially read
>     that batch or is the batch application isolated i.e. other clients
>     can only read all rows of that batch or none of them?
>
>     I understand that a BATCH update to multiple partitions is not
>     isolated but I'm not sure if this is also the case for a single
>     partition:
>
>     - The article Atomic batches in Cassandra 1.2
>     <http://www.datastax.com/dev/blog/atomic-batches-in-cassandra-1-2>
>     says that /"... we mean atomic in the database sense that if any
>     part of the batch succeeds, all of it will. No other guarantees
>     are implied; in particular, there is no isolation"/.
>
>     - On the other hand, the CQL BATCH
>     <https://cassandra.apache.org/doc/cql3/CQL.html#batchStmt> docs at
>     cassandra.apache.org <http://cassandra.apache.org> mention that
>     /"/ /... the [batch] operations are still only isolated within a
>     single partition"/ which is a clear statement but doesn't it
>     contradict the previous and the next one?
>
>     - The CQL BATCH
>     <http://docs.datastax.com/en/cql/3.1/cql/cql_reference/batch_r.html>
>     docs at docs.datastax.com <http://docs.datastax.com> mention that
>     /"... there is no batch isolation. Clients are able to read the
>     first updated rows from the batch, while other rows are still
>     being updated on the server. However, transactional row updates
>     within a partition key are isolated: clients cannot read a partial
>     update"/. Also, what does /"transactional row updates"/ mean in
>     this context? A lightweight transaction? Something else?
>
>     Thanks for any help,
>     Martin
>
>

-- 
Martin Krasser

blog:    http://krasserm.github.io
code:    http://github.com/krasserm
twitter: http://twitter.com/mrt1nz


Re: Batch isolation within a single partition

Posted by DuyHai Doan <do...@gmail.com>.
Hello Martin

If, and only if you have RF=1, single partition mutations (including
batches) are isolated.

Otherwise, with RF>1, even a simple UPDATE is not isolated because one
client can read the updated value on one replica and another client reads
the old value on another replica



On Mon, May 18, 2015 at 12:32 PM, Martin Krasser <kr...@googlemail.com>
wrote:

>  Hello,
>
> I have an application that inserts multiple rows within a single partition
> (= all rows share the same partition key) using a BATCH statement. Is it
> possible that other clients can partially read that batch or is the batch
> application isolated i.e. other clients can only read all rows of that
> batch or none of them?
>
> I understand that a BATCH update to multiple partitions is not isolated
> but I'm not sure if this is also the case for a single partition:
>
> - The article Atomic batches in Cassandra 1.2
> <http://www.datastax.com/dev/blog/atomic-batches-in-cassandra-1-2> says
> that *"... we mean atomic in the database sense that if any part of the
> batch succeeds, all of it will. No other guarantees are implied; in
> particular, there is no isolation"*.
>
> - On the other hand, the CQL BATCH
> <https://cassandra.apache.org/doc/cql3/CQL.html#batchStmt> docs at
> cassandra.apache.org mention that *"* *... the [batch] operations are
> still only isolated within a single partition"* which is a clear
> statement but doesn't it contradict the previous and the next one?
>
> - The CQL BATCH
> <http://docs.datastax.com/en/cql/3.1/cql/cql_reference/batch_r.html> docs
> at docs.datastax.com mention that *"... there is no batch isolation.
> Clients are able to read the first updated rows from the batch, while other
> rows are still being updated on the server. However, transactional row
> updates within a partition key are isolated: clients cannot read a partial
> update"*. Also, what does *"transactional row updates"* mean in this
> context? A lightweight transaction? Something else?
>
> Thanks for any help,
> Martin
>
>