You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@hbase.apache.org by Vladimir Rodionov <vl...@gmail.com> on 2014/09/12 00:58:18 UTC

HBase reads, isolation levels and RegionScanner internal locking

Hi, all

We have two isolation levels in (used to be in Scan) in Query now. See:
https://issues.apache.org/jira/browse/HBASE-11936

I moved isolation levels API from Scan upward to Query class. The reason:
this API was not available for Get operations. The rationals? Improve
performance of get and multi-gets over the same region.

As many of you aware, RegionScannerImpl is heavily synchronized on internal
region's lock.  Now some questions:

1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
mode?
We will do all necessary checks, of course, before calling nextRaw().
2. What was the reason of this locking in a first place for reads in
READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
tell me what else bad can happen?

-Vladimir

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by lars hofhansl <la...@apache.org>.
Thanks Michael, for your "RDBMS school[ing]". (Did I mention I used to work at various RDBMS companies before I came to HBase?)

Vladimir, to answer your question:
- HBase *always* locks a row for writes. Other writes to the same row will queue behind this lock.
- READ_[UN]COMITTED here only refers the whether one can see the result of prior inflight MVCC transactions. It does not affect the need for the per row write lock.
- The MVCC transactions in HBase are strictly serialize (which allows for a really simple and elegant implementations, that is valid as long as each individual transaction is short)
- READ_UNCOMMITTED will allow a client to see a partially updated row. It has no performence benefit as such, you just can see the results of other transactions earlier.
- HBase also has various region-level internal (JVM level) read and write locks that never outlive an RPC request, such as HRegion.lock and HRegion.updatesLock

I assume you refer to the latter region level locking...?

All updates (put, append, increment, delete, etc) take a *read* lock on the updatesLock to guard against concurrent flushes (which takes out a write lock). You want this one.
Whenever a region operation is started we take out a read lock on HRegion.lock to guard against concurrent bulk file operations on that region. This might be a lock we can remove with some refactoring.

HBase never locks a row for read. (It does take out some internal locks for the duration of an RPC for internal management, but a row itself is never locked for read. And certainly not across RPC requests.)

Does that make sense?

-- Lars


----- Original Message -----
From: Michael Segel <mi...@hotmail.com>
To: dev@hbase.apache.org
Cc: 
Sent: Friday, September 12, 2014 10:17 AM
Subject: Re: HBase reads, isolation levels and RegionScanner internal locking

Vlad, 

I understand. 
However several of the HBase committers aren’t really schooled in RDBMS design.

And again, the older (going back to 0.23 ) use of the term RLL isn’t relational RLL and when you start to talk about isolation you’re getting in to the RDBMS RLL 

So you really need to define what you mean when you say RLL. I don’t want to assume one thing when you meant another. 

Just like talking about salts.  ;-) 





On Sep 12, 2014, at 5:53 PM, Vladimir Rodionov <vl...@gmail.com> wrote:

> Michael, this is HBase developers mailing list.
> 
> -Vladimir
> 
> 
> 
> 
> On Fri, Sep 12, 2014 at 12:08 AM, Michael Segel <mi...@hotmail.com>
> wrote:
> 
>> Silly question…
>> 
>> HBase uses the term RLL (row level locking) to make the writes to a row
>> atomic.
>> 
>> When you start to get in to isolation, RLL takes on a different meaning.
>> 
>> So now you have to better define what do you mean by locking. Are you
>> taking about HBase RLL,
>> or are you talking about Transactional RLL ( RDBMS RLL) ?
>> 
>> 
>> On Sep 11, 2014, at 11:58 PM, Vladimir Rodionov <vl...@gmail.com>
>> wrote:
>> 
>>> Hi, all
>>> 
>>> We have two isolation levels in (used to be in Scan) in Query now. See:
>>> https://issues.apache.org/jira/browse/HBASE-11936
>>> 
>>> I moved isolation levels API from Scan upward to Query class. The reason:
>>> this API was not available for Get operations. The rationals? Improve
>>> performance of get and multi-gets over the same region.
>>> 
>>> As many of you aware, RegionScannerImpl is heavily synchronized on
>> internal
>>> region's lock.  Now some questions:
>>> 
>>> 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
>>> mode?
>>> We will do all necessary checks, of course, before calling nextRaw().
>>> 2. What was the reason of this locking in a first place for reads in
>>> READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
>>> tell me what else bad can happen?
>>> 
>>> -Vladimir
>> 
>> 


Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Michael Segel <mi...@hotmail.com>.
Vlad, 

I understand. 
However several of the HBase committers aren’t really schooled in RDBMS design.

And again, the older (going back to 0.23 ) use of the term RLL isn’t relational RLL and when you start to talk about isolation you’re getting in to the RDBMS RLL 

So you really need to define what you mean when you say RLL. I don’t want to assume one thing when you meant another. 

Just like talking about salts.  ;-) 


On Sep 12, 2014, at 5:53 PM, Vladimir Rodionov <vl...@gmail.com> wrote:

> Michael, this is HBase developers mailing list.
> 
> -Vladimir
> 
> 
> 
> 
> On Fri, Sep 12, 2014 at 12:08 AM, Michael Segel <mi...@hotmail.com>
> wrote:
> 
>> Silly question…
>> 
>> HBase uses the term RLL (row level locking) to make the writes to a row
>> atomic.
>> 
>> When you start to get in to isolation, RLL takes on a different meaning.
>> 
>> So now you have to better define what do you mean by locking. Are you
>> taking about HBase RLL,
>> or are you talking about Transactional RLL ( RDBMS RLL) ?
>> 
>> 
>> On Sep 11, 2014, at 11:58 PM, Vladimir Rodionov <vl...@gmail.com>
>> wrote:
>> 
>>> Hi, all
>>> 
>>> We have two isolation levels in (used to be in Scan) in Query now. See:
>>> https://issues.apache.org/jira/browse/HBASE-11936
>>> 
>>> I moved isolation levels API from Scan upward to Query class. The reason:
>>> this API was not available for Get operations. The rationals? Improve
>>> performance of get and multi-gets over the same region.
>>> 
>>> As many of you aware, RegionScannerImpl is heavily synchronized on
>> internal
>>> region's lock.  Now some questions:
>>> 
>>> 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
>>> mode?
>>> We will do all necessary checks, of course, before calling nextRaw().
>>> 2. What was the reason of this locking in a first place for reads in
>>> READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
>>> tell me what else bad can happen?
>>> 
>>> -Vladimir
>> 
>> 


Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Vladimir Rodionov <vl...@gmail.com>.
Michael, this is HBase developers mailing list.

-Vladimir




On Fri, Sep 12, 2014 at 12:08 AM, Michael Segel <mi...@hotmail.com>
wrote:

> Silly question…
>
> HBase uses the term RLL (row level locking) to make the writes to a row
> atomic.
>
> When you start to get in to isolation, RLL takes on a different meaning.
>
> So now you have to better define what do you mean by locking. Are you
> taking about HBase RLL,
> or are you talking about Transactional RLL ( RDBMS RLL) ?
>
>
> On Sep 11, 2014, at 11:58 PM, Vladimir Rodionov <vl...@gmail.com>
> wrote:
>
> > Hi, all
> >
> > We have two isolation levels in (used to be in Scan) in Query now. See:
> > https://issues.apache.org/jira/browse/HBASE-11936
> >
> > I moved isolation levels API from Scan upward to Query class. The reason:
> > this API was not available for Get operations. The rationals? Improve
> > performance of get and multi-gets over the same region.
> >
> > As many of you aware, RegionScannerImpl is heavily synchronized on
> internal
> > region's lock.  Now some questions:
> >
> > 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
> > mode?
> > We will do all necessary checks, of course, before calling nextRaw().
> > 2. What was the reason of this locking in a first place for reads in
> > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
> > tell me what else bad can happen?
> >
> > -Vladimir
>
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Michael Segel <mi...@hotmail.com>.
Silly question… 

HBase uses the term RLL (row level locking) to make the writes to a row atomic. 

When you start to get in to isolation, RLL takes on a different meaning. 

So now you have to better define what do you mean by locking. Are you taking about HBase RLL, 
or are you talking about Transactional RLL ( RDBMS RLL) ? 


On Sep 11, 2014, at 11:58 PM, Vladimir Rodionov <vl...@gmail.com> wrote:

> Hi, all
> 
> We have two isolation levels in (used to be in Scan) in Query now. See:
> https://issues.apache.org/jira/browse/HBASE-11936
> 
> I moved isolation levels API from Scan upward to Query class. The reason:
> this API was not available for Get operations. The rationals? Improve
> performance of get and multi-gets over the same region.
> 
> As many of you aware, RegionScannerImpl is heavily synchronized on internal
> region's lock.  Now some questions:
> 
> 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
> mode?
> We will do all necessary checks, of course, before calling nextRaw().
> 2. What was the reason of this locking in a first place for reads in
> READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
> tell me what else bad can happen?
> 
> -Vladimir


Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Vladimir Rodionov <vl...@gmail.com>.
There is no RLL (row level locking) in HBase , all locks- are region-wide.
This is the issue I am going to address
in a separate JIRA.

Example from HRegion.append

    // Lock row

    startRegionOperation(Operation.APPEND);

Regards,
-Vladimir


On Fri, Sep 12, 2014 at 11:57 AM, Vladimir Rodionov <vl...@gmail.com>
wrote:

> According to Bible:
> http://hbase.apache.org/acid-semantics.html
>
> HBase declares that all row reads are consistent - partial reads for rows
> are not possible.
>
> [
> Consistency and Isolation
>
>    1. All rows returned via any access API will consist of a complete row
>    that existed at some point in the table's history.
>    2. This is true across column families - i.e a get of a full row that
>    occurs concurrent with some mutations 1,2,3,4,5 will return a complete row
>    that existed at some point in time between mutation i and i+1 for some i
>    between 1 and 5.
>    3. The state of a row will only move forward through the history of
>    edits to it.
>
> ]
>
> In this case, there is no much sense in READ_UNCOMMITTED per se.
>
> -Vladimir
>
>
> On Fri, Sep 12, 2014 at 11:04 AM, Stack <st...@duboce.net> wrote:
>
>> On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <
>> vladrodionov@gmail.com>
>> wrote:
>>
>> > Hi, all
>> >
>> > We have two isolation levels in (used to be in Scan) in Query now. See:
>> > https://issues.apache.org/jira/browse/HBASE-11936
>> >
>> > I moved isolation levels API from Scan upward to Query class. The
>> reason:
>> > this API was not available for Get operations. The rationals? Improve
>> > performance of get and multi-gets over the same region.
>> >
>> > As many of you aware, RegionScannerImpl is heavily synchronized on
>> internal
>> > region's lock.  Now some questions:
>> >
>> > 1. Is it safe to bypass this locking (in next() call) in
>> READ_UNCOMMITTED
>> > mode?
>>
>> We will do all necessary checks, of course, before calling nextRaw().
>> > 2. What was the reason of this locking in a first place for reads in
>> > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can
>> someone
>> > tell me what else bad can happen?
>> >
>>
>>
>> There is only the obvious (that I know of) Vladimir.  We've been so
>> fixated
>> on ensuring consistent view on a row, we've not done the work to allow
>> other read types. I'm not sure what would happen if you were to skirt row
>> lock.  Try hacking on TestAtomicOperation to undo lock and see what
>> happens?
>>
>> St.Ack
>>
>
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Vladimir Rodionov <vl...@gmail.com>.
According to Bible:
http://hbase.apache.org/acid-semantics.html

HBase declares that all row reads are consistent - partial reads for rows
are not possible.

[
Consistency and Isolation

   1. All rows returned via any access API will consist of a complete row
   that existed at some point in the table's history.
   2. This is true across column families - i.e a get of a full row that
   occurs concurrent with some mutations 1,2,3,4,5 will return a complete row
   that existed at some point in time between mutation i and i+1 for some i
   between 1 and 5.
   3. The state of a row will only move forward through the history of
   edits to it.

]

In this case, there is no much sense in READ_UNCOMMITTED per se.

-Vladimir


On Fri, Sep 12, 2014 at 11:04 AM, Stack <st...@duboce.net> wrote:

> On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <vladrodionov@gmail.com
> >
> wrote:
>
> > Hi, all
> >
> > We have two isolation levels in (used to be in Scan) in Query now. See:
> > https://issues.apache.org/jira/browse/HBASE-11936
> >
> > I moved isolation levels API from Scan upward to Query class. The reason:
> > this API was not available for Get operations. The rationals? Improve
> > performance of get and multi-gets over the same region.
> >
> > As many of you aware, RegionScannerImpl is heavily synchronized on
> internal
> > region's lock.  Now some questions:
> >
> > 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
> > mode?
>
> We will do all necessary checks, of course, before calling nextRaw().
> > 2. What was the reason of this locking in a first place for reads in
> > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
> > tell me what else bad can happen?
> >
>
>
> There is only the obvious (that I know of) Vladimir.  We've been so fixated
> on ensuring consistent view on a row, we've not done the work to allow
> other read types. I'm not sure what would happen if you were to skirt row
> lock.  Try hacking on TestAtomicOperation to undo lock and see what
> happens?
>
> St.Ack
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Michael Segel <mi...@hotmail.com>.
Atomic is atomic and that’s not an issue. 

But if you’re locking a row… what are you locking? 

The HFiles that make up the row? The CFs? The memory? 
How long do you hold the lock(s)? 


On Sep 12, 2014, at 9:29 PM, Stack <st...@duboce.net> wrote:

> On Fri, Sep 12, 2014 at 1:21 PM, Vladimir Rodionov <vl...@gmail.com>
> wrote:
> 
>> All row mutate operations in HBase are atomic:
>> puts/deletes/increments/appends.
>> Atomicity in HBase has the same meaning as in RDBMs exactly - operations
>> completes as a whole or does not at all.  There is an additional guarantee
>> in HBase that all reads are ROW - atomic as well - one will never read
>> partial result of atomic mutate operation (on a ROW). READ_UNCOMMITTED will
>> weaken this read atomicity guarantee and it will be possible to read
>> partial results of a row mutation operation,
>> 
>> so we discard row read atomicity but still have cell atomicity
>> (hopefully?).
>> 
>> This is my understanding of what READ_UNCOMMITTED means in HBase.
>> 
>> PS
>> 
>> I created :
>> https://issues.apache.org/jira/browse/HBASE-11965
> 
> 
> Thanks Vladimir,
> St.Ack


Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Stack <st...@duboce.net>.
On Fri, Sep 12, 2014 at 1:21 PM, Vladimir Rodionov <vl...@gmail.com>
wrote:

> All row mutate operations in HBase are atomic:
> puts/deletes/increments/appends.
> Atomicity in HBase has the same meaning as in RDBMs exactly - operations
> completes as a whole or does not at all.  There is an additional guarantee
> in HBase that all reads are ROW - atomic as well - one will never read
> partial result of atomic mutate operation (on a ROW). READ_UNCOMMITTED will
> weaken this read atomicity guarantee and it will be possible to read
> partial results of a row mutation operation,
>
> so we discard row read atomicity but still have cell atomicity
> (hopefully?).
>
> This is my understanding of what READ_UNCOMMITTED means in HBase.
>
> PS
>
> I created :
> https://issues.apache.org/jira/browse/HBASE-11965


Thanks Vladimir,
St.Ack

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Vladimir Rodionov <vl...@gmail.com>.
All row mutate operations in HBase are atomic: puts/deletes/increments/appends.
Atomicity in HBase has the same meaning as in RDBMs exactly - operations
completes as a whole or does not at all.  There is an additional guarantee
in HBase that all reads are ROW - atomic as well - one will never read
partial result of atomic mutate operation (on a ROW). READ_UNCOMMITTED will
weaken this read atomicity guarantee and it will be possible to read
partial results of a row mutation operation,

so we discard row read atomicity but still have cell atomicity (hopefully?).

This is my understanding of what READ_UNCOMMITTED means in HBase.

PS

I created :
https://issues.apache.org/jira/browse/HBASE-11965

-Vladimir


On Fri, Sep 12, 2014 at 12:28 PM, Stack <st...@duboce.net> wrote:

> On Fri, Sep 12, 2014 at 11:25 AM, Vladimir Rodionov <
> vladrodionov@gmail.com>
> wrote:
>
> > Michael,
> >
> > This is not a row-level locking - it is region-wide lock. This is a major
> > reason of  the following performance problems:
> >
> >
> Pardon my misreading as row-scoping (I'd just come off reading Michael
> Segel's note).
>
>
>
> > 1) Multi gets are bad if inside the same region
> > 2) Multiple scanners over the same region are bad
> > 3) Scan during compaction are bad.
> >
>
>
>
> > I need some input from HBase folks here:
> >
> > 1) READ_UNCOMMITTED safe if lock free?
> >
>
>
> And rely on MVCC only? That'd be cool Vladimir.  When you say
> READ_UNCOMMITTED, are you row or cell-scoped?  Are you thinking that you'd
> make it so the row lock was also optional?
>
>
>
> > 2) Confirmation that region-wide lock is for read consistency only.
> >
> >
> >
> The region lock, IIRC, was added so we can close the region cleanly.  All
> gets/puts/etc. take the lock and hold it while operating to prevent the
> region being closed out from under them.  It was put in place long ago and
> not revisited since.
>
> Hope this helps.
>
>
> Related, there is Liang Xie's effort over in HDFS:
> https://issues.apache.org/jira/browse/HDFS-6735
> St.Ack.
>
>
>
> > On Fri, Sep 12, 2014 at 11:04 AM, Stack <st...@duboce.net> wrote:
> >
> > > On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <
> > vladrodionov@gmail.com
> > > >
> > > wrote:
> > >
> > > > Hi, all
> > > >
> > > > We have two isolation levels in (used to be in Scan) in Query now.
> See:
> > > > https://issues.apache.org/jira/browse/HBASE-11936
> > > >
> > > > I moved isolation levels API from Scan upward to Query class. The
> > reason:
> > > > this API was not available for Get operations. The rationals? Improve
> > > > performance of get and multi-gets over the same region.
> > > >
> > > > As many of you aware, RegionScannerImpl is heavily synchronized on
> > > internal
> > > > region's lock.  Now some questions:
> > > >
> > > > 1. Is it safe to bypass this locking (in next() call) in
> > READ_UNCOMMITTED
> > > > mode?
> > >
> > > We will do all necessary checks, of course, before calling nextRaw().
> > > > 2. What was the reason of this locking in a first place for reads in
> > > > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can
> > someone
> > > > tell me what else bad can happen?
> > > >
> > >
> > >
> > > There is only the obvious (that I know of) Vladimir.  We've been so
> > fixated
> > > on ensuring consistent view on a row, we've not done the work to allow
> > > other read types. I'm not sure what would happen if you were to skirt
> row
> > > lock.  Try hacking on TestAtomicOperation to undo lock and see what
> > > happens?
> > >
> > > St.Ack
> > >
> >
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Stack <st...@duboce.net>.
On Fri, Sep 12, 2014 at 11:25 AM, Vladimir Rodionov <vl...@gmail.com>
wrote:

> Michael,
>
> This is not a row-level locking - it is region-wide lock. This is a major
> reason of  the following performance problems:
>
>
Pardon my misreading as row-scoping (I'd just come off reading Michael
Segel's note).



> 1) Multi gets are bad if inside the same region
> 2) Multiple scanners over the same region are bad
> 3) Scan during compaction are bad.
>



> I need some input from HBase folks here:
>
> 1) READ_UNCOMMITTED safe if lock free?
>


And rely on MVCC only? That'd be cool Vladimir.  When you say
READ_UNCOMMITTED, are you row or cell-scoped?  Are you thinking that you'd
make it so the row lock was also optional?



> 2) Confirmation that region-wide lock is for read consistency only.
>
>
>
The region lock, IIRC, was added so we can close the region cleanly.  All
gets/puts/etc. take the lock and hold it while operating to prevent the
region being closed out from under them.  It was put in place long ago and
not revisited since.

Hope this helps.


Related, there is Liang Xie's effort over in HDFS:
https://issues.apache.org/jira/browse/HDFS-6735
St.Ack.



> On Fri, Sep 12, 2014 at 11:04 AM, Stack <st...@duboce.net> wrote:
>
> > On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <
> vladrodionov@gmail.com
> > >
> > wrote:
> >
> > > Hi, all
> > >
> > > We have two isolation levels in (used to be in Scan) in Query now. See:
> > > https://issues.apache.org/jira/browse/HBASE-11936
> > >
> > > I moved isolation levels API from Scan upward to Query class. The
> reason:
> > > this API was not available for Get operations. The rationals? Improve
> > > performance of get and multi-gets over the same region.
> > >
> > > As many of you aware, RegionScannerImpl is heavily synchronized on
> > internal
> > > region's lock.  Now some questions:
> > >
> > > 1. Is it safe to bypass this locking (in next() call) in
> READ_UNCOMMITTED
> > > mode?
> >
> > We will do all necessary checks, of course, before calling nextRaw().
> > > 2. What was the reason of this locking in a first place for reads in
> > > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can
> someone
> > > tell me what else bad can happen?
> > >
> >
> >
> > There is only the obvious (that I know of) Vladimir.  We've been so
> fixated
> > on ensuring consistent view on a row, we've not done the work to allow
> > other read types. I'm not sure what would happen if you were to skirt row
> > lock.  Try hacking on TestAtomicOperation to undo lock and see what
> > happens?
> >
> > St.Ack
> >
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Vladimir Rodionov <vl...@gmail.com>.
Michael,

This is not a row-level locking - it is region-wide lock. This is a major
reason of  the following performance problems:

1) Multi gets are bad if inside the same region
2) Multiple scanners over the same region are bad
3) Scan during compaction are bad.

I need some input from HBase folks here:

1) READ_UNCOMMITTED safe if lock free?
2) Confirmation that region-wide lock is for read consistency only.


On Fri, Sep 12, 2014 at 11:04 AM, Stack <st...@duboce.net> wrote:

> On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <vladrodionov@gmail.com
> >
> wrote:
>
> > Hi, all
> >
> > We have two isolation levels in (used to be in Scan) in Query now. See:
> > https://issues.apache.org/jira/browse/HBASE-11936
> >
> > I moved isolation levels API from Scan upward to Query class. The reason:
> > this API was not available for Get operations. The rationals? Improve
> > performance of get and multi-gets over the same region.
> >
> > As many of you aware, RegionScannerImpl is heavily synchronized on
> internal
> > region's lock.  Now some questions:
> >
> > 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
> > mode?
>
> We will do all necessary checks, of course, before calling nextRaw().
> > 2. What was the reason of this locking in a first place for reads in
> > READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
> > tell me what else bad can happen?
> >
>
>
> There is only the obvious (that I know of) Vladimir.  We've been so fixated
> on ensuring consistent view on a row, we've not done the work to allow
> other read types. I'm not sure what would happen if you were to skirt row
> lock.  Try hacking on TestAtomicOperation to undo lock and see what
> happens?
>
> St.Ack
>

Re: HBase reads, isolation levels and RegionScanner internal locking

Posted by Stack <st...@duboce.net>.
On Thu, Sep 11, 2014 at 3:58 PM, Vladimir Rodionov <vl...@gmail.com>
wrote:

> Hi, all
>
> We have two isolation levels in (used to be in Scan) in Query now. See:
> https://issues.apache.org/jira/browse/HBASE-11936
>
> I moved isolation levels API from Scan upward to Query class. The reason:
> this API was not available for Get operations. The rationals? Improve
> performance of get and multi-gets over the same region.
>
> As many of you aware, RegionScannerImpl is heavily synchronized on internal
> region's lock.  Now some questions:
>
> 1. Is it safe to bypass this locking (in next() call) in READ_UNCOMMITTED
> mode?

We will do all necessary checks, of course, before calling nextRaw().
> 2. What was the reason of this locking in a first place for reads in
> READ_COMMITTED mode? Except obvious - no-dirty-reads allowed? Can someone
> tell me what else bad can happen?
>


There is only the obvious (that I know of) Vladimir.  We've been so fixated
on ensuring consistent view on a row, we've not done the work to allow
other read types. I'm not sure what would happen if you were to skirt row
lock.  Try hacking on TestAtomicOperation to undo lock and see what happens?

St.Ack