You are viewing a plain text version of this content. The canonical link for it is here.
Posted to user@hbase.apache.org by Jianshi Huang <ji...@gmail.com> on 2014/09/05 17:15:35 UTC

Help: RegionTooBusyException: failed to get a lock in 60000 ms

I'm importing 2TB of generated HFiles to HBase and I constantly get the
following errors:

Caused by:
org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.hbase.RegionTooBusyException):
org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock in
60000 ms.
regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f3565642d96.,
server=xxxxx.xxx.xxx,60020,1404854700728
        at
org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
        at
org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
        at
org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRegion.java:5795)
        at
org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3543)
        at
org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3525)
        at
org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionServer.java:3277)
        at
org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.callBlockingMethod(ClientProtos.java:28863)
        at org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
        at org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
        at
org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcScheduler.java:160)
        at
org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcScheduler.java:38)
        at
org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.java:110)
        at java.lang.Thread.run(Thread.java:724)

        at org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
        at
org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1684)
        at
org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.callBlockingMethod(RpcClient.java:1737)
        at
org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$BlockingStub.bulkLoadHFile(ClientProtos.java:29276)
        at
org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.java:1548)
        ... 11 more


What makes the region too busy? Is there a way to improve it?

Does that also mean some part of my data are not correctly imported?


Thanks,

-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
See HBASE-10882 where Victor's workaround was to do major compaction before
the bulkload.

Cheers


On Fri, Sep 5, 2014 at 8:15 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> I'm importing 2TB of generated HFiles to HBase and I constantly get the
> following errors:
>
> Caused by:
>
> org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.hbase.RegionTooBusyException):
> org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock in
> 60000 ms.
>
> regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f3565642d96.,
> server=xxxxx.xxx.xxx,60020,1404854700728
>         at
> org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
>         at
> org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRegion.java:5795)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3543)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3525)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionServer.java:3277)
>         at
>
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.callBlockingMethod(ClientProtos.java:28863)
>         at org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
>         at org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcScheduler.java:160)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcScheduler.java:38)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.java:110)
>         at java.lang.Thread.run(Thread.java:724)
>
>         at org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
>         at
>
> org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1684)
>         at
>
> org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.callBlockingMethod(RpcClient.java:1737)
>         at
>
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$BlockingStub.bulkLoadHFile(ClientProtos.java:29276)
>         at
>
> org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.java:1548)
>         ... 11 more
>
>
> What makes the region too busy? Is there a way to improve it?
>
> Does that also mean some part of my data are not correctly imported?
>
>
> Thanks,
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
Cycling previous bits (w.r.t. adjusting weights for table skew):
http://search-hadoop.com/m/DHED4CWSqW1/snapshot+timeout+problem&subj=Re+snapshot+timeout+problem

Cheers

On Mon, Sep 22, 2014 at 8:35 PM, Qiang Tian <ti...@gmail.com> wrote:

> Hello, I happened to got balancer related issues 2 months ago and looked at
> that part, below is a summary:
> 1)by default, hbase balancer(StochasticLoadBalancer by default) does not
> balance regions per table. i.e. all regions are considered as 1 table.  so
> if you have many tables, especially some tables have empty regions, you
> probably get unbalanced, the balancer probably not triggered at all.
> this is got from code inspection, my problem failed to be reproduced later.
> but it proved that deleting empty regions can trigger balancer correctly
> and make regions well balanced.
>
> 2)there are some other reasons that balancer are not triggered. see
> HMaster#balance. turn on debug can see related messages in master log. in
> my case, it is not triggered because there are regions in transition:
> LOG.debug("Not running balancer because " + regionsInTransition.size() +
>           " region(s) in transition: " +
> org.apache.commons.lang.StringUtils.
>             abbreviate(regionsInTransition.toString(), 256));
>
> the cause can be found in regionserver log file.
>
> 3)per-table balance can be set by "hbase.master.loadbalance.bytable",
> however it looks not a good option when you have many tables - the master
> will issue balance call for each table, one by one.
>
> 4)split region follows normal balancer process. so if you have issue in #1,
> split does not help balance.  it looks pre-split at table creation is fine,
> which uses round-robin assignment.
>
>
>
> On Tue, Sep 23, 2014 at 2:12 AM, Bharath Vissapragada <
> bharathv@cloudera.com
> > wrote:
>
> > https://issues.apache.org/jira/browse/HBASE-11368 related to the
> original
> > issue too.
> >
> > On Mon, Sep 22, 2014 at 10:18 AM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > As you noted in the FIXME, there're some factors which should be
> tackled
> > by
> > > balancer / assignment manager.
> > >
> > > Please continue digging up master log so that we can find the cause for
> > > balancer not fulfilling your goal.
> > >
> > > Cheers
> > >
> > > On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > wrote:
> > >
> > > > Ok, I fixed this by manually reassign region servers to newly created
> > > ones.
> > > >
> > > >   def reassignRegionServer(admin: HBaseAdmin, regions:
> > Seq[HRegionInfo],
> > > > regionServers: Seq[ServerName]): Unit = {
> > > >     val rand = new Random()
> > > >     regions.foreach { r =>
> > > >       val idx = rand.nextInt(regionServers.size)
> > > >       val server = regionServers(idx)
> > > >       // FIXME: what if selected region server is dead?
> > > >         admin.move(r.getEncodedNameAsBytes,
> > > > server.getServerName.getBytes("UTF8"))
> > > >     }
> > > >   }
> > > >
> > > > er...
> > > >
> > > > Jianshi
> > > >
> > > > On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <
> > jianshi.huang@gmail.com
> > > >
> > > > wrote:
> > > >
> > > > > Hmm...any workaround? I only want to do this:
> > > > >
> > > > > Rebalance the new regions *evenly* to all servers after manually
> > adding
> > > > > splits, so later bulk insertions won't cause contention.
> > > > >
> > > > > P.S.
> > > > > Looks like two of the region servers which had majority of the
> > regions
> > > > > were down during Major compaction... I guess it had too much data.
> > > > >
> > > > >
> > > > > Jianshi
> > > > >
> > > > > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com
> > > > >
> > > > > wrote:
> > > > >
> > > > >> Yes, I have access to Master UI, however logs/*.log cannot be
> opened
> > > or
> > > > >> downloaded, must be some security restrictions in the proxy...
> > > > >>
> > > > >> Jianshi
> > > > >>
> > > > >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com>
> > wrote:
> > > > >>
> > > > >>> Do you have access to Master UI ?
> > > > >>>
> > > > >>> <master-address>:60010/logs/ would show you list of log files.
> > > > >>>
> > > > >>> The you can view
> > > > <master-address>:60010/logs/hbase-<user>-master-XXX.log
> > > > >>>
> > > > >>> Cheers
> > > > >>>
> > > > >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
> > > > jianshi.huang@gmail.com>
> > > > >>> wrote:
> > > > >>>
> > > > >>> > Ah... I don't have access to HMaster logs... I need to ask the
> > > admin.
> > > > >>> >
> > > > >>> > Jianshi
> > > > >>> >
> > > > >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
> > > > wrote:
> > > > >>> >
> > > > >>> > > bq. assign per-table balancer class
> > > > >>> > >
> > > > >>> > > No that I know of.
> > > > >>> > > Can you pastebin master log involving output from balancer ?
> > > > >>> > >
> > > > >>> > > Cheers
> > > > >>> > >
> > > > >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> > > > >>> jianshi.huang@gmail.com>
> > > > >>> > > wrote:
> > > > >>> > >
> > > > >>> > > > Hi Ted,
> > > > >>> > > >
> > > > >>> > > > I moved setBalancerRunning before balancer and run them
> > twice.
> > > > >>> However
> > > > >>> > I
> > > > >>> > > > still got highly skewed region distribution.
> > > > >>> > > >
> > > > >>> > > > I guess it's because of the StochasticLoadBalancer, can I
> > > assign
> > > > >>> > > per-table
> > > > >>> > > > balancer class in HBase?
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > > Jianshi
> > > > >>> > > >
> > > > >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <
> yuzhihong@gmail.com
> > >
> > > > >>> wrote:
> > > > >>> > > >
> > > > >>> > > > > admin.setBalancerRunning() call should precede the call
> to
> > > > >>> > > > > admin.balancer().
> > > > >>> > > > >
> > > > >>> > > > > You can inspect master log to see whether regions are
> being
> > > > >>> moved off
> > > > >>> > > the
> > > > >>> > > > > heavily loaded server.
> > > > >>> > > > >
> > > > >>> > > > > Cheers
> > > > >>> > > > >
> > > > >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > > > >>> > > jianshi.huang@gmail.com>
> > > > >>> > > > > wrote:
> > > > >>> > > > >
> > > > >>> > > > > > Hi Ted and others,
> > > > >>> > > > > >
> > > > >>> > > > > > I did the following after adding splits (without data)
> to
> > > my
> > > > >>> table,
> > > > >>> > > > > however
> > > > >>> > > > > > the region is still very imbalanced (one region server
> > has
> > > > 221
> > > > >>> > > regions
> > > > >>> > > > > and
> > > > >>> > > > > > other 50 region servers have about 4~8 regions each).
> > > > >>> > > > > >
> > > > >>> > > > > >       admin.balancer()
> > > > >>> > > > > >       admin.setBalancerRunning(true, true)
> > > > >>> > > > > >
> > > > >>> > > > > > The balancer class in my HBase cluster is
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > > >>> > > > > >
> > > > >>> > > > > > So, is this behavior expected? Can I assign different
> > > > balancer
> > > > >>> > class
> > > > >>> > > to
> > > > >>> > > > > my
> > > > >>> > > > > > tables (I don't have HBase admin permission)? Which one
> > > > should
> > > > >>> I
> > > > >>> > use?
> > > > >>> > > > > >
> > > > >>> > > > > > I just want HBase to evenly distribute the regions even
> > > they
> > > > >>> don't
> > > > >>> > > have
> > > > >>> > > > > > data (that's the purpose of pre-split I think).
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > Jianshi
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <
> > > yuzhihong@gmail.com
> > > > >
> > > > >>> > wrote:
> > > > >>> > > > > >
> > > > >>> > > > > > > Yes. See the following method in HBaseAdmin:
> > > > >>> > > > > > >
> > > > >>> > > > > > >   public boolean balancer()
> > > > >>> > > > > > >
> > > > >>> > > > > > >
> > > > >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > > > >>> > > > jianshi.huang@gmail.com
> > > > >>> > > > > >
> > > > >>> > > > > > > wrote:
> > > > >>> > > > > > >
> > > > >>> > > > > > > > Thanks Ted!
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Didn't know I still need to run the 'balancer'
> > command.
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Is there a way to do it programmatically?
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Jianshi
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
> > > > >>> yuzhihong@gmail.com>
> > > > >>> > > > wrote:
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > > After splitting the region, you may need to run
> > > > balancer
> > > > >>> to
> > > > >>> > > > spread
> > > > >>> > > > > > the
> > > > >>> > > > > > > > new
> > > > >>> > > > > > > > > regions out.
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > Cheers
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > > >>> > > > > > jianshi.huang@gmail.com
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > > wrote:
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > > Hi Shahab,
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > I see, that seems to be the right way...
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > >>> > > > > > > shahab.yunus@gmail.com>
> > > > >>> > > > > > > > > > wrote:
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > > Shahab
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > --
> > > > >>> > > > > > > > > > Jianshi Huang
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > LinkedIn: jianshi
> > > > >>> > > > > > > > > > Twitter: @jshuang
> > > > >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > --
> > > > >>> > > > > > > > Jianshi Huang
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > LinkedIn: jianshi
> > > > >>> > > > > > > > Twitter: @jshuang
> > > > >>> > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > > > >
> > > > >>> > > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > --
> > > > >>> > > > > > Jianshi Huang
> > > > >>> > > > > >
> > > > >>> > > > > > LinkedIn: jianshi
> > > > >>> > > > > > Twitter: @jshuang
> > > > >>> > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > >
> > > > >>> > > > >
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > > --
> > > > >>> > > > Jianshi Huang
> > > > >>> > > >
> > > > >>> > > > LinkedIn: jianshi
> > > > >>> > > > Twitter: @jshuang
> > > > >>> > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > >
> > > > >>> > >
> > > > >>> >
> > > > >>> >
> > > > >>> >
> > > > >>> > --
> > > > >>> > Jianshi Huang
> > > > >>> >
> > > > >>> > LinkedIn: jianshi
> > > > >>> > Twitter: @jshuang
> > > > >>> > Github & Blog: http://huangjs.github.com/
> > > > >>> >
> > > > >>>
> > > > >>
> > > > >>
> > > > >>
> > > > >> --
> > > > >> Jianshi Huang
> > > > >>
> > > > >> LinkedIn: jianshi
> > > > >> Twitter: @jshuang
> > > > >> Github & Blog: http://huangjs.github.com/
> > > > >>
> > > > >
> > > > >
> > > > >
> > > > > --
> > > > > Jianshi Huang
> > > > >
> > > > > LinkedIn: jianshi
> > > > > Twitter: @jshuang
> > > > > Github & Blog: http://huangjs.github.com/
> > > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Bharath Vissapragada
> > <http://www.cloudera.com>
> >
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Qiang Tian <ti...@gmail.com>.
considering 80-20 rule(does it apply to hbase tables?), what about
specifying balance property(per-table vs non per-table) on a table
basis(default is non per-table), then those few large tables could be
balanced well with acceptable total balance cost?


On Tue, Sep 23, 2014 at 11:35 AM, Qiang Tian <ti...@gmail.com> wrote:

> Hello, I happened to got balancer related issues 2 months ago and looked
> at that part, below is a summary:
> 1)by default, hbase balancer(StochasticLoadBalancer by default) does not
> balance regions per table. i.e. all regions are considered as 1 table.  so
> if you have many tables, especially some tables have empty regions, you
> probably get unbalanced, the balancer probably not triggered at all.
> this is got from code inspection, my problem failed to be reproduced
> later. but it proved that deleting empty regions can trigger balancer
> correctly and make regions well balanced.
>
> 2)there are some other reasons that balancer are not triggered. see
> HMaster#balance. turn on debug can see related messages in master log. in
> my case, it is not triggered because there are regions in transition:
> LOG.debug("Not running balancer because " + regionsInTransition.size() +
>           " region(s) in transition: " +
> org.apache.commons.lang.StringUtils.
>             abbreviate(regionsInTransition.toString(), 256));
>
> the cause can be found in regionserver log file.
>
> 3)per-table balance can be set by "hbase.master.loadbalance.bytable",
> however it looks not a good option when you have many tables - the master
> will issue balance call for each table, one by one.
>
> 4)split region follows normal balancer process. so if you have issue in
> #1, split does not help balance.  it looks pre-split at table creation is
> fine, which uses round-robin assignment.
>
>
>
> On Tue, Sep 23, 2014 at 2:12 AM, Bharath Vissapragada <
> bharathv@cloudera.com> wrote:
>
>> https://issues.apache.org/jira/browse/HBASE-11368 related to the original
>> issue too.
>>
>> On Mon, Sep 22, 2014 at 10:18 AM, Ted Yu <yu...@gmail.com> wrote:
>>
>> > As you noted in the FIXME, there're some factors which should be
>> tackled by
>> > balancer / assignment manager.
>> >
>> > Please continue digging up master log so that we can find the cause for
>> > balancer not fulfilling your goal.
>> >
>> > Cheers
>> >
>> > On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <
>> jianshi.huang@gmail.com>
>> > wrote:
>> >
>> > > Ok, I fixed this by manually reassign region servers to newly created
>> > ones.
>> > >
>> > >   def reassignRegionServer(admin: HBaseAdmin, regions:
>> Seq[HRegionInfo],
>> > > regionServers: Seq[ServerName]): Unit = {
>> > >     val rand = new Random()
>> > >     regions.foreach { r =>
>> > >       val idx = rand.nextInt(regionServers.size)
>> > >       val server = regionServers(idx)
>> > >       // FIXME: what if selected region server is dead?
>> > >         admin.move(r.getEncodedNameAsBytes,
>> > > server.getServerName.getBytes("UTF8"))
>> > >     }
>> > >   }
>> > >
>> > > er...
>> > >
>> > > Jianshi
>> > >
>> > > On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <
>> jianshi.huang@gmail.com
>> > >
>> > > wrote:
>> > >
>> > > > Hmm...any workaround? I only want to do this:
>> > > >
>> > > > Rebalance the new regions *evenly* to all servers after manually
>> adding
>> > > > splits, so later bulk insertions won't cause contention.
>> > > >
>> > > > P.S.
>> > > > Looks like two of the region servers which had majority of the
>> regions
>> > > > were down during Major compaction... I guess it had too much data.
>> > > >
>> > > >
>> > > > Jianshi
>> > > >
>> > > > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <
>> > jianshi.huang@gmail.com
>> > > >
>> > > > wrote:
>> > > >
>> > > >> Yes, I have access to Master UI, however logs/*.log cannot be
>> opened
>> > or
>> > > >> downloaded, must be some security restrictions in the proxy...
>> > > >>
>> > > >> Jianshi
>> > > >>
>> > > >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com>
>> wrote:
>> > > >>
>> > > >>> Do you have access to Master UI ?
>> > > >>>
>> > > >>> <master-address>:60010/logs/ would show you list of log files.
>> > > >>>
>> > > >>> The you can view
>> > > <master-address>:60010/logs/hbase-<user>-master-XXX.log
>> > > >>>
>> > > >>> Cheers
>> > > >>>
>> > > >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
>> > > jianshi.huang@gmail.com>
>> > > >>> wrote:
>> > > >>>
>> > > >>> > Ah... I don't have access to HMaster logs... I need to ask the
>> > admin.
>> > > >>> >
>> > > >>> > Jianshi
>> > > >>> >
>> > > >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
>> > > wrote:
>> > > >>> >
>> > > >>> > > bq. assign per-table balancer class
>> > > >>> > >
>> > > >>> > > No that I know of.
>> > > >>> > > Can you pastebin master log involving output from balancer ?
>> > > >>> > >
>> > > >>> > > Cheers
>> > > >>> > >
>> > > >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
>> > > >>> jianshi.huang@gmail.com>
>> > > >>> > > wrote:
>> > > >>> > >
>> > > >>> > > > Hi Ted,
>> > > >>> > > >
>> > > >>> > > > I moved setBalancerRunning before balancer and run them
>> twice.
>> > > >>> However
>> > > >>> > I
>> > > >>> > > > still got highly skewed region distribution.
>> > > >>> > > >
>> > > >>> > > > I guess it's because of the StochasticLoadBalancer, can I
>> > assign
>> > > >>> > > per-table
>> > > >>> > > > balancer class in HBase?
>> > > >>> > > >
>> > > >>> > > >
>> > > >>> > > > Jianshi
>> > > >>> > > >
>> > > >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <
>> yuzhihong@gmail.com>
>> > > >>> wrote:
>> > > >>> > > >
>> > > >>> > > > > admin.setBalancerRunning() call should precede the call to
>> > > >>> > > > > admin.balancer().
>> > > >>> > > > >
>> > > >>> > > > > You can inspect master log to see whether regions are
>> being
>> > > >>> moved off
>> > > >>> > > the
>> > > >>> > > > > heavily loaded server.
>> > > >>> > > > >
>> > > >>> > > > > Cheers
>> > > >>> > > > >
>> > > >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
>> > > >>> > > jianshi.huang@gmail.com>
>> > > >>> > > > > wrote:
>> > > >>> > > > >
>> > > >>> > > > > > Hi Ted and others,
>> > > >>> > > > > >
>> > > >>> > > > > > I did the following after adding splits (without data)
>> to
>> > my
>> > > >>> table,
>> > > >>> > > > > however
>> > > >>> > > > > > the region is still very imbalanced (one region server
>> has
>> > > 221
>> > > >>> > > regions
>> > > >>> > > > > and
>> > > >>> > > > > > other 50 region servers have about 4~8 regions each).
>> > > >>> > > > > >
>> > > >>> > > > > >       admin.balancer()
>> > > >>> > > > > >       admin.setBalancerRunning(true, true)
>> > > >>> > > > > >
>> > > >>> > > > > > The balancer class in my HBase cluster is
>> > > >>> > > > > >
>> > > >>> > > > > >
>> > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
>> > > >>> > > > > >
>> > > >>> > > > > > So, is this behavior expected? Can I assign different
>> > > balancer
>> > > >>> > class
>> > > >>> > > to
>> > > >>> > > > > my
>> > > >>> > > > > > tables (I don't have HBase admin permission)? Which one
>> > > should
>> > > >>> I
>> > > >>> > use?
>> > > >>> > > > > >
>> > > >>> > > > > > I just want HBase to evenly distribute the regions even
>> > they
>> > > >>> don't
>> > > >>> > > have
>> > > >>> > > > > > data (that's the purpose of pre-split I think).
>> > > >>> > > > > >
>> > > >>> > > > > >
>> > > >>> > > > > > Jianshi
>> > > >>> > > > > >
>> > > >>> > > > > >
>> > > >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <
>> > yuzhihong@gmail.com
>> > > >
>> > > >>> > wrote:
>> > > >>> > > > > >
>> > > >>> > > > > > > Yes. See the following method in HBaseAdmin:
>> > > >>> > > > > > >
>> > > >>> > > > > > >   public boolean balancer()
>> > > >>> > > > > > >
>> > > >>> > > > > > >
>> > > >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
>> > > >>> > > > jianshi.huang@gmail.com
>> > > >>> > > > > >
>> > > >>> > > > > > > wrote:
>> > > >>> > > > > > >
>> > > >>> > > > > > > > Thanks Ted!
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > Didn't know I still need to run the 'balancer'
>> command.
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > Is there a way to do it programmatically?
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > Jianshi
>> > > >>> > > > > > > >
>> > > >>> > > > > > > >
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
>> > > >>> yuzhihong@gmail.com>
>> > > >>> > > > wrote:
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > > After splitting the region, you may need to run
>> > > balancer
>> > > >>> to
>> > > >>> > > > spread
>> > > >>> > > > > > the
>> > > >>> > > > > > > > new
>> > > >>> > > > > > > > > regions out.
>> > > >>> > > > > > > > >
>> > > >>> > > > > > > > > Cheers
>> > > >>> > > > > > > > >
>> > > >>> > > > > > > > >
>> > > >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
>> > > >>> > > > > > jianshi.huang@gmail.com
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > > wrote:
>> > > >>> > > > > > > > >
>> > > >>> > > > > > > > > > Hi Shahab,
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > > I see, that seems to be the right way...
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
>> > > >>> > > > > > > shahab.yunus@gmail.com>
>> > > >>> > > > > > > > > > wrote:
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > > > Shahab
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > > --
>> > > >>> > > > > > > > > > Jianshi Huang
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > > > LinkedIn: jianshi
>> > > >>> > > > > > > > > > Twitter: @jshuang
>> > > >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
>> > > >>> > > > > > > > > >
>> > > >>> > > > > > > > >
>> > > >>> > > > > > > >
>> > > >>> > > > > > > >
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > --
>> > > >>> > > > > > > > Jianshi Huang
>> > > >>> > > > > > > >
>> > > >>> > > > > > > > LinkedIn: jianshi
>> > > >>> > > > > > > > Twitter: @jshuang
>> > > >>> > > > > > > > Github & Blog: http://huangjs.github.com/
>> > > >>> > > > > > > >
>> > > >>> > > > > > >
>> > > >>> > > > > >
>> > > >>> > > > > >
>> > > >>> > > > > >
>> > > >>> > > > > > --
>> > > >>> > > > > > Jianshi Huang
>> > > >>> > > > > >
>> > > >>> > > > > > LinkedIn: jianshi
>> > > >>> > > > > > Twitter: @jshuang
>> > > >>> > > > > > Github & Blog: http://huangjs.github.com/
>> > > >>> > > > > >
>> > > >>> > > > >
>> > > >>> > > >
>> > > >>> > > >
>> > > >>> > > >
>> > > >>> > > > --
>> > > >>> > > > Jianshi Huang
>> > > >>> > > >
>> > > >>> > > > LinkedIn: jianshi
>> > > >>> > > > Twitter: @jshuang
>> > > >>> > > > Github & Blog: http://huangjs.github.com/
>> > > >>> > > >
>> > > >>> > >
>> > > >>> >
>> > > >>> >
>> > > >>> >
>> > > >>> > --
>> > > >>> > Jianshi Huang
>> > > >>> >
>> > > >>> > LinkedIn: jianshi
>> > > >>> > Twitter: @jshuang
>> > > >>> > Github & Blog: http://huangjs.github.com/
>> > > >>> >
>> > > >>>
>> > > >>
>> > > >>
>> > > >>
>> > > >> --
>> > > >> Jianshi Huang
>> > > >>
>> > > >> LinkedIn: jianshi
>> > > >> Twitter: @jshuang
>> > > >> Github & Blog: http://huangjs.github.com/
>> > > >>
>> > > >
>> > > >
>> > > >
>> > > > --
>> > > > Jianshi Huang
>> > > >
>> > > > LinkedIn: jianshi
>> > > > Twitter: @jshuang
>> > > > Github & Blog: http://huangjs.github.com/
>> > > >
>> > >
>> > >
>> > >
>> > > --
>> > > Jianshi Huang
>> > >
>> > > LinkedIn: jianshi
>> > > Twitter: @jshuang
>> > > Github & Blog: http://huangjs.github.com/
>> > >
>> >
>>
>>
>>
>> --
>> Bharath Vissapragada
>> <http://www.cloudera.com>
>>
>
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Thanks for the explanation Qian!

I think being able to balance empty regions is important and the preferred
result to me.

The best way so far is to manually 'balance' the regions if we need to add
pre-splits dynamically.


Jianshi



On Tue, Sep 23, 2014 at 11:35 AM, Qiang Tian <ti...@gmail.com> wrote:

> Hello, I happened to got balancer related issues 2 months ago and looked at
> that part, below is a summary:
> 1)by default, hbase balancer(StochasticLoadBalancer by default) does not
> balance regions per table. i.e. all regions are considered as 1 table.  so
> if you have many tables, especially some tables have empty regions, you
> probably get unbalanced, the balancer probably not triggered at all.
> this is got from code inspection, my problem failed to be reproduced later.
> but it proved that deleting empty regions can trigger balancer correctly
> and make regions well balanced.
>
> 2)there are some other reasons that balancer are not triggered. see
> HMaster#balance. turn on debug can see related messages in master log. in
> my case, it is not triggered because there are regions in transition:
> LOG.debug("Not running balancer because " + regionsInTransition.size() +
>           " region(s) in transition: " +
> org.apache.commons.lang.StringUtils.
>             abbreviate(regionsInTransition.toString(), 256));
>
> the cause can be found in regionserver log file.
>
> 3)per-table balance can be set by "hbase.master.loadbalance.bytable",
> however it looks not a good option when you have many tables - the master
> will issue balance call for each table, one by one.
>
> 4)split region follows normal balancer process. so if you have issue in #1,
> split does not help balance.  it looks pre-split at table creation is fine,
> which uses round-robin assignment.
>
>
>
> On Tue, Sep 23, 2014 at 2:12 AM, Bharath Vissapragada <
> bharathv@cloudera.com
> > wrote:
>
> > https://issues.apache.org/jira/browse/HBASE-11368 related to the
> original
> > issue too.
> >
> > On Mon, Sep 22, 2014 at 10:18 AM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > As you noted in the FIXME, there're some factors which should be
> tackled
> > by
> > > balancer / assignment manager.
> > >
> > > Please continue digging up master log so that we can find the cause for
> > > balancer not fulfilling your goal.
> > >
> > > Cheers
> > >
> > > On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > wrote:
> > >
> > > > Ok, I fixed this by manually reassign region servers to newly created
> > > ones.
> > > >
> > > >   def reassignRegionServer(admin: HBaseAdmin, regions:
> > Seq[HRegionInfo],
> > > > regionServers: Seq[ServerName]): Unit = {
> > > >     val rand = new Random()
> > > >     regions.foreach { r =>
> > > >       val idx = rand.nextInt(regionServers.size)
> > > >       val server = regionServers(idx)
> > > >       // FIXME: what if selected region server is dead?
> > > >         admin.move(r.getEncodedNameAsBytes,
> > > > server.getServerName.getBytes("UTF8"))
> > > >     }
> > > >   }
> > > >
> > > > er...
> > > >
> > > > Jianshi
> > > >
> > > > On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <
> > jianshi.huang@gmail.com
> > > >
> > > > wrote:
> > > >
> > > > > Hmm...any workaround? I only want to do this:
> > > > >
> > > > > Rebalance the new regions *evenly* to all servers after manually
> > adding
> > > > > splits, so later bulk insertions won't cause contention.
> > > > >
> > > > > P.S.
> > > > > Looks like two of the region servers which had majority of the
> > regions
> > > > > were down during Major compaction... I guess it had too much data.
> > > > >
> > > > >
> > > > > Jianshi
> > > > >
> > > > > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com
> > > > >
> > > > > wrote:
> > > > >
> > > > >> Yes, I have access to Master UI, however logs/*.log cannot be
> opened
> > > or
> > > > >> downloaded, must be some security restrictions in the proxy...
> > > > >>
> > > > >> Jianshi
> > > > >>
> > > > >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com>
> > wrote:
> > > > >>
> > > > >>> Do you have access to Master UI ?
> > > > >>>
> > > > >>> <master-address>:60010/logs/ would show you list of log files.
> > > > >>>
> > > > >>> The you can view
> > > > <master-address>:60010/logs/hbase-<user>-master-XXX.log
> > > > >>>
> > > > >>> Cheers
> > > > >>>
> > > > >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
> > > > jianshi.huang@gmail.com>
> > > > >>> wrote:
> > > > >>>
> > > > >>> > Ah... I don't have access to HMaster logs... I need to ask the
> > > admin.
> > > > >>> >
> > > > >>> > Jianshi
> > > > >>> >
> > > > >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
> > > > wrote:
> > > > >>> >
> > > > >>> > > bq. assign per-table balancer class
> > > > >>> > >
> > > > >>> > > No that I know of.
> > > > >>> > > Can you pastebin master log involving output from balancer ?
> > > > >>> > >
> > > > >>> > > Cheers
> > > > >>> > >
> > > > >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> > > > >>> jianshi.huang@gmail.com>
> > > > >>> > > wrote:
> > > > >>> > >
> > > > >>> > > > Hi Ted,
> > > > >>> > > >
> > > > >>> > > > I moved setBalancerRunning before balancer and run them
> > twice.
> > > > >>> However
> > > > >>> > I
> > > > >>> > > > still got highly skewed region distribution.
> > > > >>> > > >
> > > > >>> > > > I guess it's because of the StochasticLoadBalancer, can I
> > > assign
> > > > >>> > > per-table
> > > > >>> > > > balancer class in HBase?
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > > Jianshi
> > > > >>> > > >
> > > > >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <
> yuzhihong@gmail.com
> > >
> > > > >>> wrote:
> > > > >>> > > >
> > > > >>> > > > > admin.setBalancerRunning() call should precede the call
> to
> > > > >>> > > > > admin.balancer().
> > > > >>> > > > >
> > > > >>> > > > > You can inspect master log to see whether regions are
> being
> > > > >>> moved off
> > > > >>> > > the
> > > > >>> > > > > heavily loaded server.
> > > > >>> > > > >
> > > > >>> > > > > Cheers
> > > > >>> > > > >
> > > > >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > > > >>> > > jianshi.huang@gmail.com>
> > > > >>> > > > > wrote:
> > > > >>> > > > >
> > > > >>> > > > > > Hi Ted and others,
> > > > >>> > > > > >
> > > > >>> > > > > > I did the following after adding splits (without data)
> to
> > > my
> > > > >>> table,
> > > > >>> > > > > however
> > > > >>> > > > > > the region is still very imbalanced (one region server
> > has
> > > > 221
> > > > >>> > > regions
> > > > >>> > > > > and
> > > > >>> > > > > > other 50 region servers have about 4~8 regions each).
> > > > >>> > > > > >
> > > > >>> > > > > >       admin.balancer()
> > > > >>> > > > > >       admin.setBalancerRunning(true, true)
> > > > >>> > > > > >
> > > > >>> > > > > > The balancer class in my HBase cluster is
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > > >>> > > > > >
> > > > >>> > > > > > So, is this behavior expected? Can I assign different
> > > > balancer
> > > > >>> > class
> > > > >>> > > to
> > > > >>> > > > > my
> > > > >>> > > > > > tables (I don't have HBase admin permission)? Which one
> > > > should
> > > > >>> I
> > > > >>> > use?
> > > > >>> > > > > >
> > > > >>> > > > > > I just want HBase to evenly distribute the regions even
> > > they
> > > > >>> don't
> > > > >>> > > have
> > > > >>> > > > > > data (that's the purpose of pre-split I think).
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > Jianshi
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <
> > > yuzhihong@gmail.com
> > > > >
> > > > >>> > wrote:
> > > > >>> > > > > >
> > > > >>> > > > > > > Yes. See the following method in HBaseAdmin:
> > > > >>> > > > > > >
> > > > >>> > > > > > >   public boolean balancer()
> > > > >>> > > > > > >
> > > > >>> > > > > > >
> > > > >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > > > >>> > > > jianshi.huang@gmail.com
> > > > >>> > > > > >
> > > > >>> > > > > > > wrote:
> > > > >>> > > > > > >
> > > > >>> > > > > > > > Thanks Ted!
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Didn't know I still need to run the 'balancer'
> > command.
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Is there a way to do it programmatically?
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > Jianshi
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
> > > > >>> yuzhihong@gmail.com>
> > > > >>> > > > wrote:
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > > After splitting the region, you may need to run
> > > > balancer
> > > > >>> to
> > > > >>> > > > spread
> > > > >>> > > > > > the
> > > > >>> > > > > > > > new
> > > > >>> > > > > > > > > regions out.
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > Cheers
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > > >>> > > > > > jianshi.huang@gmail.com
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > > wrote:
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > > > > Hi Shahab,
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > I see, that seems to be the right way...
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > >>> > > > > > > shahab.yunus@gmail.com>
> > > > >>> > > > > > > > > > wrote:
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > > Shahab
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > --
> > > > >>> > > > > > > > > > Jianshi Huang
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > > > LinkedIn: jianshi
> > > > >>> > > > > > > > > > Twitter: @jshuang
> > > > >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > > > > > >
> > > > >>> > > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > --
> > > > >>> > > > > > > > Jianshi Huang
> > > > >>> > > > > > > >
> > > > >>> > > > > > > > LinkedIn: jianshi
> > > > >>> > > > > > > > Twitter: @jshuang
> > > > >>> > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > > > >
> > > > >>> > > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > >
> > > > >>> > > > > > --
> > > > >>> > > > > > Jianshi Huang
> > > > >>> > > > > >
> > > > >>> > > > > > LinkedIn: jianshi
> > > > >>> > > > > > Twitter: @jshuang
> > > > >>> > > > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > > > >
> > > > >>> > > > >
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > >
> > > > >>> > > > --
> > > > >>> > > > Jianshi Huang
> > > > >>> > > >
> > > > >>> > > > LinkedIn: jianshi
> > > > >>> > > > Twitter: @jshuang
> > > > >>> > > > Github & Blog: http://huangjs.github.com/
> > > > >>> > > >
> > > > >>> > >
> > > > >>> >
> > > > >>> >
> > > > >>> >
> > > > >>> > --
> > > > >>> > Jianshi Huang
> > > > >>> >
> > > > >>> > LinkedIn: jianshi
> > > > >>> > Twitter: @jshuang
> > > > >>> > Github & Blog: http://huangjs.github.com/
> > > > >>> >
> > > > >>>
> > > > >>
> > > > >>
> > > > >>
> > > > >> --
> > > > >> Jianshi Huang
> > > > >>
> > > > >> LinkedIn: jianshi
> > > > >> Twitter: @jshuang
> > > > >> Github & Blog: http://huangjs.github.com/
> > > > >>
> > > > >
> > > > >
> > > > >
> > > > > --
> > > > > Jianshi Huang
> > > > >
> > > > > LinkedIn: jianshi
> > > > > Twitter: @jshuang
> > > > > Github & Blog: http://huangjs.github.com/
> > > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Bharath Vissapragada
> > <http://www.cloudera.com>
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Qiang Tian <ti...@gmail.com>.
Hello, I happened to got balancer related issues 2 months ago and looked at
that part, below is a summary:
1)by default, hbase balancer(StochasticLoadBalancer by default) does not
balance regions per table. i.e. all regions are considered as 1 table.  so
if you have many tables, especially some tables have empty regions, you
probably get unbalanced, the balancer probably not triggered at all.
this is got from code inspection, my problem failed to be reproduced later.
but it proved that deleting empty regions can trigger balancer correctly
and make regions well balanced.

2)there are some other reasons that balancer are not triggered. see
HMaster#balance. turn on debug can see related messages in master log. in
my case, it is not triggered because there are regions in transition:
LOG.debug("Not running balancer because " + regionsInTransition.size() +
          " region(s) in transition: " +
org.apache.commons.lang.StringUtils.
            abbreviate(regionsInTransition.toString(), 256));

the cause can be found in regionserver log file.

3)per-table balance can be set by "hbase.master.loadbalance.bytable",
however it looks not a good option when you have many tables - the master
will issue balance call for each table, one by one.

4)split region follows normal balancer process. so if you have issue in #1,
split does not help balance.  it looks pre-split at table creation is fine,
which uses round-robin assignment.



On Tue, Sep 23, 2014 at 2:12 AM, Bharath Vissapragada <bharathv@cloudera.com
> wrote:

> https://issues.apache.org/jira/browse/HBASE-11368 related to the original
> issue too.
>
> On Mon, Sep 22, 2014 at 10:18 AM, Ted Yu <yu...@gmail.com> wrote:
>
> > As you noted in the FIXME, there're some factors which should be tackled
> by
> > balancer / assignment manager.
> >
> > Please continue digging up master log so that we can find the cause for
> > balancer not fulfilling your goal.
> >
> > Cheers
> >
> > On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <jianshi.huang@gmail.com
> >
> > wrote:
> >
> > > Ok, I fixed this by manually reassign region servers to newly created
> > ones.
> > >
> > >   def reassignRegionServer(admin: HBaseAdmin, regions:
> Seq[HRegionInfo],
> > > regionServers: Seq[ServerName]): Unit = {
> > >     val rand = new Random()
> > >     regions.foreach { r =>
> > >       val idx = rand.nextInt(regionServers.size)
> > >       val server = regionServers(idx)
> > >       // FIXME: what if selected region server is dead?
> > >         admin.move(r.getEncodedNameAsBytes,
> > > server.getServerName.getBytes("UTF8"))
> > >     }
> > >   }
> > >
> > > er...
> > >
> > > Jianshi
> > >
> > > On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > wrote:
> > >
> > > > Hmm...any workaround? I only want to do this:
> > > >
> > > > Rebalance the new regions *evenly* to all servers after manually
> adding
> > > > splits, so later bulk insertions won't cause contention.
> > > >
> > > > P.S.
> > > > Looks like two of the region servers which had majority of the
> regions
> > > > were down during Major compaction... I guess it had too much data.
> > > >
> > > >
> > > > Jianshi
> > > >
> > > > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <
> > jianshi.huang@gmail.com
> > > >
> > > > wrote:
> > > >
> > > >> Yes, I have access to Master UI, however logs/*.log cannot be opened
> > or
> > > >> downloaded, must be some security restrictions in the proxy...
> > > >>
> > > >> Jianshi
> > > >>
> > > >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com>
> wrote:
> > > >>
> > > >>> Do you have access to Master UI ?
> > > >>>
> > > >>> <master-address>:60010/logs/ would show you list of log files.
> > > >>>
> > > >>> The you can view
> > > <master-address>:60010/logs/hbase-<user>-master-XXX.log
> > > >>>
> > > >>> Cheers
> > > >>>
> > > >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com>
> > > >>> wrote:
> > > >>>
> > > >>> > Ah... I don't have access to HMaster logs... I need to ask the
> > admin.
> > > >>> >
> > > >>> > Jianshi
> > > >>> >
> > > >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
> > > wrote:
> > > >>> >
> > > >>> > > bq. assign per-table balancer class
> > > >>> > >
> > > >>> > > No that I know of.
> > > >>> > > Can you pastebin master log involving output from balancer ?
> > > >>> > >
> > > >>> > > Cheers
> > > >>> > >
> > > >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> > > >>> jianshi.huang@gmail.com>
> > > >>> > > wrote:
> > > >>> > >
> > > >>> > > > Hi Ted,
> > > >>> > > >
> > > >>> > > > I moved setBalancerRunning before balancer and run them
> twice.
> > > >>> However
> > > >>> > I
> > > >>> > > > still got highly skewed region distribution.
> > > >>> > > >
> > > >>> > > > I guess it's because of the StochasticLoadBalancer, can I
> > assign
> > > >>> > > per-table
> > > >>> > > > balancer class in HBase?
> > > >>> > > >
> > > >>> > > >
> > > >>> > > > Jianshi
> > > >>> > > >
> > > >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yuzhihong@gmail.com
> >
> > > >>> wrote:
> > > >>> > > >
> > > >>> > > > > admin.setBalancerRunning() call should precede the call to
> > > >>> > > > > admin.balancer().
> > > >>> > > > >
> > > >>> > > > > You can inspect master log to see whether regions are being
> > > >>> moved off
> > > >>> > > the
> > > >>> > > > > heavily loaded server.
> > > >>> > > > >
> > > >>> > > > > Cheers
> > > >>> > > > >
> > > >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > > >>> > > jianshi.huang@gmail.com>
> > > >>> > > > > wrote:
> > > >>> > > > >
> > > >>> > > > > > Hi Ted and others,
> > > >>> > > > > >
> > > >>> > > > > > I did the following after adding splits (without data) to
> > my
> > > >>> table,
> > > >>> > > > > however
> > > >>> > > > > > the region is still very imbalanced (one region server
> has
> > > 221
> > > >>> > > regions
> > > >>> > > > > and
> > > >>> > > > > > other 50 region servers have about 4~8 regions each).
> > > >>> > > > > >
> > > >>> > > > > >       admin.balancer()
> > > >>> > > > > >       admin.setBalancerRunning(true, true)
> > > >>> > > > > >
> > > >>> > > > > > The balancer class in my HBase cluster is
> > > >>> > > > > >
> > > >>> > > > > >
> > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > >>> > > > > >
> > > >>> > > > > > So, is this behavior expected? Can I assign different
> > > balancer
> > > >>> > class
> > > >>> > > to
> > > >>> > > > > my
> > > >>> > > > > > tables (I don't have HBase admin permission)? Which one
> > > should
> > > >>> I
> > > >>> > use?
> > > >>> > > > > >
> > > >>> > > > > > I just want HBase to evenly distribute the regions even
> > they
> > > >>> don't
> > > >>> > > have
> > > >>> > > > > > data (that's the purpose of pre-split I think).
> > > >>> > > > > >
> > > >>> > > > > >
> > > >>> > > > > > Jianshi
> > > >>> > > > > >
> > > >>> > > > > >
> > > >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <
> > yuzhihong@gmail.com
> > > >
> > > >>> > wrote:
> > > >>> > > > > >
> > > >>> > > > > > > Yes. See the following method in HBaseAdmin:
> > > >>> > > > > > >
> > > >>> > > > > > >   public boolean balancer()
> > > >>> > > > > > >
> > > >>> > > > > > >
> > > >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > > >>> > > > jianshi.huang@gmail.com
> > > >>> > > > > >
> > > >>> > > > > > > wrote:
> > > >>> > > > > > >
> > > >>> > > > > > > > Thanks Ted!
> > > >>> > > > > > > >
> > > >>> > > > > > > > Didn't know I still need to run the 'balancer'
> command.
> > > >>> > > > > > > >
> > > >>> > > > > > > > Is there a way to do it programmatically?
> > > >>> > > > > > > >
> > > >>> > > > > > > > Jianshi
> > > >>> > > > > > > >
> > > >>> > > > > > > >
> > > >>> > > > > > > >
> > > >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
> > > >>> yuzhihong@gmail.com>
> > > >>> > > > wrote:
> > > >>> > > > > > > >
> > > >>> > > > > > > > > After splitting the region, you may need to run
> > > balancer
> > > >>> to
> > > >>> > > > spread
> > > >>> > > > > > the
> > > >>> > > > > > > > new
> > > >>> > > > > > > > > regions out.
> > > >>> > > > > > > > >
> > > >>> > > > > > > > > Cheers
> > > >>> > > > > > > > >
> > > >>> > > > > > > > >
> > > >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > >>> > > > > > jianshi.huang@gmail.com
> > > >>> > > > > > > >
> > > >>> > > > > > > > > wrote:
> > > >>> > > > > > > > >
> > > >>> > > > > > > > > > Hi Shahab,
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > > I see, that seems to be the right way...
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > >>> > > > > > > shahab.yunus@gmail.com>
> > > >>> > > > > > > > > > wrote:
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > > > Shahab
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > > --
> > > >>> > > > > > > > > > Jianshi Huang
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > > > LinkedIn: jianshi
> > > >>> > > > > > > > > > Twitter: @jshuang
> > > >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> > > >>> > > > > > > > > >
> > > >>> > > > > > > > >
> > > >>> > > > > > > >
> > > >>> > > > > > > >
> > > >>> > > > > > > >
> > > >>> > > > > > > > --
> > > >>> > > > > > > > Jianshi Huang
> > > >>> > > > > > > >
> > > >>> > > > > > > > LinkedIn: jianshi
> > > >>> > > > > > > > Twitter: @jshuang
> > > >>> > > > > > > > Github & Blog: http://huangjs.github.com/
> > > >>> > > > > > > >
> > > >>> > > > > > >
> > > >>> > > > > >
> > > >>> > > > > >
> > > >>> > > > > >
> > > >>> > > > > > --
> > > >>> > > > > > Jianshi Huang
> > > >>> > > > > >
> > > >>> > > > > > LinkedIn: jianshi
> > > >>> > > > > > Twitter: @jshuang
> > > >>> > > > > > Github & Blog: http://huangjs.github.com/
> > > >>> > > > > >
> > > >>> > > > >
> > > >>> > > >
> > > >>> > > >
> > > >>> > > >
> > > >>> > > > --
> > > >>> > > > Jianshi Huang
> > > >>> > > >
> > > >>> > > > LinkedIn: jianshi
> > > >>> > > > Twitter: @jshuang
> > > >>> > > > Github & Blog: http://huangjs.github.com/
> > > >>> > > >
> > > >>> > >
> > > >>> >
> > > >>> >
> > > >>> >
> > > >>> > --
> > > >>> > Jianshi Huang
> > > >>> >
> > > >>> > LinkedIn: jianshi
> > > >>> > Twitter: @jshuang
> > > >>> > Github & Blog: http://huangjs.github.com/
> > > >>> >
> > > >>>
> > > >>
> > > >>
> > > >>
> > > >> --
> > > >> Jianshi Huang
> > > >>
> > > >> LinkedIn: jianshi
> > > >> Twitter: @jshuang
> > > >> Github & Blog: http://huangjs.github.com/
> > > >>
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Bharath Vissapragada
> <http://www.cloudera.com>
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Bharath Vissapragada <bh...@cloudera.com>.
https://issues.apache.org/jira/browse/HBASE-11368 related to the original
issue too.

On Mon, Sep 22, 2014 at 10:18 AM, Ted Yu <yu...@gmail.com> wrote:

> As you noted in the FIXME, there're some factors which should be tackled by
> balancer / assignment manager.
>
> Please continue digging up master log so that we can find the cause for
> balancer not fulfilling your goal.
>
> Cheers
>
> On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Ok, I fixed this by manually reassign region servers to newly created
> ones.
> >
> >   def reassignRegionServer(admin: HBaseAdmin, regions: Seq[HRegionInfo],
> > regionServers: Seq[ServerName]): Unit = {
> >     val rand = new Random()
> >     regions.foreach { r =>
> >       val idx = rand.nextInt(regionServers.size)
> >       val server = regionServers(idx)
> >       // FIXME: what if selected region server is dead?
> >         admin.move(r.getEncodedNameAsBytes,
> > server.getServerName.getBytes("UTF8"))
> >     }
> >   }
> >
> > er...
> >
> > Jianshi
> >
> > On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <jianshi.huang@gmail.com
> >
> > wrote:
> >
> > > Hmm...any workaround? I only want to do this:
> > >
> > > Rebalance the new regions *evenly* to all servers after manually adding
> > > splits, so later bulk insertions won't cause contention.
> > >
> > > P.S.
> > > Looks like two of the region servers which had majority of the regions
> > > were down during Major compaction... I guess it had too much data.
> > >
> > >
> > > Jianshi
> > >
> > > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > wrote:
> > >
> > >> Yes, I have access to Master UI, however logs/*.log cannot be opened
> or
> > >> downloaded, must be some security restrictions in the proxy...
> > >>
> > >> Jianshi
> > >>
> > >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com> wrote:
> > >>
> > >>> Do you have access to Master UI ?
> > >>>
> > >>> <master-address>:60010/logs/ would show you list of log files.
> > >>>
> > >>> The you can view
> > <master-address>:60010/logs/hbase-<user>-master-XXX.log
> > >>>
> > >>> Cheers
> > >>>
> > >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
> > jianshi.huang@gmail.com>
> > >>> wrote:
> > >>>
> > >>> > Ah... I don't have access to HMaster logs... I need to ask the
> admin.
> > >>> >
> > >>> > Jianshi
> > >>> >
> > >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
> > wrote:
> > >>> >
> > >>> > > bq. assign per-table balancer class
> > >>> > >
> > >>> > > No that I know of.
> > >>> > > Can you pastebin master log involving output from balancer ?
> > >>> > >
> > >>> > > Cheers
> > >>> > >
> > >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> > >>> jianshi.huang@gmail.com>
> > >>> > > wrote:
> > >>> > >
> > >>> > > > Hi Ted,
> > >>> > > >
> > >>> > > > I moved setBalancerRunning before balancer and run them twice.
> > >>> However
> > >>> > I
> > >>> > > > still got highly skewed region distribution.
> > >>> > > >
> > >>> > > > I guess it's because of the StochasticLoadBalancer, can I
> assign
> > >>> > > per-table
> > >>> > > > balancer class in HBase?
> > >>> > > >
> > >>> > > >
> > >>> > > > Jianshi
> > >>> > > >
> > >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com>
> > >>> wrote:
> > >>> > > >
> > >>> > > > > admin.setBalancerRunning() call should precede the call to
> > >>> > > > > admin.balancer().
> > >>> > > > >
> > >>> > > > > You can inspect master log to see whether regions are being
> > >>> moved off
> > >>> > > the
> > >>> > > > > heavily loaded server.
> > >>> > > > >
> > >>> > > > > Cheers
> > >>> > > > >
> > >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > >>> > > jianshi.huang@gmail.com>
> > >>> > > > > wrote:
> > >>> > > > >
> > >>> > > > > > Hi Ted and others,
> > >>> > > > > >
> > >>> > > > > > I did the following after adding splits (without data) to
> my
> > >>> table,
> > >>> > > > > however
> > >>> > > > > > the region is still very imbalanced (one region server has
> > 221
> > >>> > > regions
> > >>> > > > > and
> > >>> > > > > > other 50 region servers have about 4~8 regions each).
> > >>> > > > > >
> > >>> > > > > >       admin.balancer()
> > >>> > > > > >       admin.setBalancerRunning(true, true)
> > >>> > > > > >
> > >>> > > > > > The balancer class in my HBase cluster is
> > >>> > > > > >
> > >>> > > > > >
> > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > >>> > > > > >
> > >>> > > > > > So, is this behavior expected? Can I assign different
> > balancer
> > >>> > class
> > >>> > > to
> > >>> > > > > my
> > >>> > > > > > tables (I don't have HBase admin permission)? Which one
> > should
> > >>> I
> > >>> > use?
> > >>> > > > > >
> > >>> > > > > > I just want HBase to evenly distribute the regions even
> they
> > >>> don't
> > >>> > > have
> > >>> > > > > > data (that's the purpose of pre-split I think).
> > >>> > > > > >
> > >>> > > > > >
> > >>> > > > > > Jianshi
> > >>> > > > > >
> > >>> > > > > >
> > >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <
> yuzhihong@gmail.com
> > >
> > >>> > wrote:
> > >>> > > > > >
> > >>> > > > > > > Yes. See the following method in HBaseAdmin:
> > >>> > > > > > >
> > >>> > > > > > >   public boolean balancer()
> > >>> > > > > > >
> > >>> > > > > > >
> > >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > >>> > > > jianshi.huang@gmail.com
> > >>> > > > > >
> > >>> > > > > > > wrote:
> > >>> > > > > > >
> > >>> > > > > > > > Thanks Ted!
> > >>> > > > > > > >
> > >>> > > > > > > > Didn't know I still need to run the 'balancer' command.
> > >>> > > > > > > >
> > >>> > > > > > > > Is there a way to do it programmatically?
> > >>> > > > > > > >
> > >>> > > > > > > > Jianshi
> > >>> > > > > > > >
> > >>> > > > > > > >
> > >>> > > > > > > >
> > >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
> > >>> yuzhihong@gmail.com>
> > >>> > > > wrote:
> > >>> > > > > > > >
> > >>> > > > > > > > > After splitting the region, you may need to run
> > balancer
> > >>> to
> > >>> > > > spread
> > >>> > > > > > the
> > >>> > > > > > > > new
> > >>> > > > > > > > > regions out.
> > >>> > > > > > > > >
> > >>> > > > > > > > > Cheers
> > >>> > > > > > > > >
> > >>> > > > > > > > >
> > >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > >>> > > > > > jianshi.huang@gmail.com
> > >>> > > > > > > >
> > >>> > > > > > > > > wrote:
> > >>> > > > > > > > >
> > >>> > > > > > > > > > Hi Shahab,
> > >>> > > > > > > > > >
> > >>> > > > > > > > > > I see, that seems to be the right way...
> > >>> > > > > > > > > >
> > >>> > > > > > > > > >
> > >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > >>> > > > > > > shahab.yunus@gmail.com>
> > >>> > > > > > > > > > wrote:
> > >>> > > > > > > > > >
> > >>> > > > > > > > > > > Shahab
> > >>> > > > > > > > > >
> > >>> > > > > > > > > >
> > >>> > > > > > > > > >
> > >>> > > > > > > > > >
> > >>> > > > > > > > > >
> > >>> > > > > > > > > > --
> > >>> > > > > > > > > > Jianshi Huang
> > >>> > > > > > > > > >
> > >>> > > > > > > > > > LinkedIn: jianshi
> > >>> > > > > > > > > > Twitter: @jshuang
> > >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> > >>> > > > > > > > > >
> > >>> > > > > > > > >
> > >>> > > > > > > >
> > >>> > > > > > > >
> > >>> > > > > > > >
> > >>> > > > > > > > --
> > >>> > > > > > > > Jianshi Huang
> > >>> > > > > > > >
> > >>> > > > > > > > LinkedIn: jianshi
> > >>> > > > > > > > Twitter: @jshuang
> > >>> > > > > > > > Github & Blog: http://huangjs.github.com/
> > >>> > > > > > > >
> > >>> > > > > > >
> > >>> > > > > >
> > >>> > > > > >
> > >>> > > > > >
> > >>> > > > > > --
> > >>> > > > > > Jianshi Huang
> > >>> > > > > >
> > >>> > > > > > LinkedIn: jianshi
> > >>> > > > > > Twitter: @jshuang
> > >>> > > > > > Github & Blog: http://huangjs.github.com/
> > >>> > > > > >
> > >>> > > > >
> > >>> > > >
> > >>> > > >
> > >>> > > >
> > >>> > > > --
> > >>> > > > Jianshi Huang
> > >>> > > >
> > >>> > > > LinkedIn: jianshi
> > >>> > > > Twitter: @jshuang
> > >>> > > > Github & Blog: http://huangjs.github.com/
> > >>> > > >
> > >>> > >
> > >>> >
> > >>> >
> > >>> >
> > >>> > --
> > >>> > Jianshi Huang
> > >>> >
> > >>> > LinkedIn: jianshi
> > >>> > Twitter: @jshuang
> > >>> > Github & Blog: http://huangjs.github.com/
> > >>> >
> > >>>
> > >>
> > >>
> > >>
> > >> --
> > >> Jianshi Huang
> > >>
> > >> LinkedIn: jianshi
> > >> Twitter: @jshuang
> > >> Github & Blog: http://huangjs.github.com/
> > >>
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Bharath Vissapragada
<http://www.cloudera.com>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
As you noted in the FIXME, there're some factors which should be tackled by
balancer / assignment manager.

Please continue digging up master log so that we can find the cause for
balancer not fulfilling your goal.

Cheers

On Mon, Sep 22, 2014 at 10:09 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Ok, I fixed this by manually reassign region servers to newly created ones.
>
>   def reassignRegionServer(admin: HBaseAdmin, regions: Seq[HRegionInfo],
> regionServers: Seq[ServerName]): Unit = {
>     val rand = new Random()
>     regions.foreach { r =>
>       val idx = rand.nextInt(regionServers.size)
>       val server = regionServers(idx)
>       // FIXME: what if selected region server is dead?
>         admin.move(r.getEncodedNameAsBytes,
> server.getServerName.getBytes("UTF8"))
>     }
>   }
>
> er...
>
> Jianshi
>
> On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Hmm...any workaround? I only want to do this:
> >
> > Rebalance the new regions *evenly* to all servers after manually adding
> > splits, so later bulk insertions won't cause contention.
> >
> > P.S.
> > Looks like two of the region servers which had majority of the regions
> > were down during Major compaction... I guess it had too much data.
> >
> >
> > Jianshi
> >
> > On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <jianshi.huang@gmail.com
> >
> > wrote:
> >
> >> Yes, I have access to Master UI, however logs/*.log cannot be opened or
> >> downloaded, must be some security restrictions in the proxy...
> >>
> >> Jianshi
> >>
> >> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com> wrote:
> >>
> >>> Do you have access to Master UI ?
> >>>
> >>> <master-address>:60010/logs/ would show you list of log files.
> >>>
> >>> The you can view
> <master-address>:60010/logs/hbase-<user>-master-XXX.log
> >>>
> >>> Cheers
> >>>
> >>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <
> jianshi.huang@gmail.com>
> >>> wrote:
> >>>
> >>> > Ah... I don't have access to HMaster logs... I need to ask the admin.
> >>> >
> >>> > Jianshi
> >>> >
> >>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com>
> wrote:
> >>> >
> >>> > > bq. assign per-table balancer class
> >>> > >
> >>> > > No that I know of.
> >>> > > Can you pastebin master log involving output from balancer ?
> >>> > >
> >>> > > Cheers
> >>> > >
> >>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> >>> jianshi.huang@gmail.com>
> >>> > > wrote:
> >>> > >
> >>> > > > Hi Ted,
> >>> > > >
> >>> > > > I moved setBalancerRunning before balancer and run them twice.
> >>> However
> >>> > I
> >>> > > > still got highly skewed region distribution.
> >>> > > >
> >>> > > > I guess it's because of the StochasticLoadBalancer, can I assign
> >>> > > per-table
> >>> > > > balancer class in HBase?
> >>> > > >
> >>> > > >
> >>> > > > Jianshi
> >>> > > >
> >>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com>
> >>> wrote:
> >>> > > >
> >>> > > > > admin.setBalancerRunning() call should precede the call to
> >>> > > > > admin.balancer().
> >>> > > > >
> >>> > > > > You can inspect master log to see whether regions are being
> >>> moved off
> >>> > > the
> >>> > > > > heavily loaded server.
> >>> > > > >
> >>> > > > > Cheers
> >>> > > > >
> >>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> >>> > > jianshi.huang@gmail.com>
> >>> > > > > wrote:
> >>> > > > >
> >>> > > > > > Hi Ted and others,
> >>> > > > > >
> >>> > > > > > I did the following after adding splits (without data) to my
> >>> table,
> >>> > > > > however
> >>> > > > > > the region is still very imbalanced (one region server has
> 221
> >>> > > regions
> >>> > > > > and
> >>> > > > > > other 50 region servers have about 4~8 regions each).
> >>> > > > > >
> >>> > > > > >       admin.balancer()
> >>> > > > > >       admin.setBalancerRunning(true, true)
> >>> > > > > >
> >>> > > > > > The balancer class in my HBase cluster is
> >>> > > > > >
> >>> > > > > >
> org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> >>> > > > > >
> >>> > > > > > So, is this behavior expected? Can I assign different
> balancer
> >>> > class
> >>> > > to
> >>> > > > > my
> >>> > > > > > tables (I don't have HBase admin permission)? Which one
> should
> >>> I
> >>> > use?
> >>> > > > > >
> >>> > > > > > I just want HBase to evenly distribute the regions even they
> >>> don't
> >>> > > have
> >>> > > > > > data (that's the purpose of pre-split I think).
> >>> > > > > >
> >>> > > > > >
> >>> > > > > > Jianshi
> >>> > > > > >
> >>> > > > > >
> >>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yuzhihong@gmail.com
> >
> >>> > wrote:
> >>> > > > > >
> >>> > > > > > > Yes. See the following method in HBaseAdmin:
> >>> > > > > > >
> >>> > > > > > >   public boolean balancer()
> >>> > > > > > >
> >>> > > > > > >
> >>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> >>> > > > jianshi.huang@gmail.com
> >>> > > > > >
> >>> > > > > > > wrote:
> >>> > > > > > >
> >>> > > > > > > > Thanks Ted!
> >>> > > > > > > >
> >>> > > > > > > > Didn't know I still need to run the 'balancer' command.
> >>> > > > > > > >
> >>> > > > > > > > Is there a way to do it programmatically?
> >>> > > > > > > >
> >>> > > > > > > > Jianshi
> >>> > > > > > > >
> >>> > > > > > > >
> >>> > > > > > > >
> >>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
> >>> yuzhihong@gmail.com>
> >>> > > > wrote:
> >>> > > > > > > >
> >>> > > > > > > > > After splitting the region, you may need to run
> balancer
> >>> to
> >>> > > > spread
> >>> > > > > > the
> >>> > > > > > > > new
> >>> > > > > > > > > regions out.
> >>> > > > > > > > >
> >>> > > > > > > > > Cheers
> >>> > > > > > > > >
> >>> > > > > > > > >
> >>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> >>> > > > > > jianshi.huang@gmail.com
> >>> > > > > > > >
> >>> > > > > > > > > wrote:
> >>> > > > > > > > >
> >>> > > > > > > > > > Hi Shahab,
> >>> > > > > > > > > >
> >>> > > > > > > > > > I see, that seems to be the right way...
> >>> > > > > > > > > >
> >>> > > > > > > > > >
> >>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> >>> > > > > > > shahab.yunus@gmail.com>
> >>> > > > > > > > > > wrote:
> >>> > > > > > > > > >
> >>> > > > > > > > > > > Shahab
> >>> > > > > > > > > >
> >>> > > > > > > > > >
> >>> > > > > > > > > >
> >>> > > > > > > > > >
> >>> > > > > > > > > >
> >>> > > > > > > > > > --
> >>> > > > > > > > > > Jianshi Huang
> >>> > > > > > > > > >
> >>> > > > > > > > > > LinkedIn: jianshi
> >>> > > > > > > > > > Twitter: @jshuang
> >>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> >>> > > > > > > > > >
> >>> > > > > > > > >
> >>> > > > > > > >
> >>> > > > > > > >
> >>> > > > > > > >
> >>> > > > > > > > --
> >>> > > > > > > > Jianshi Huang
> >>> > > > > > > >
> >>> > > > > > > > LinkedIn: jianshi
> >>> > > > > > > > Twitter: @jshuang
> >>> > > > > > > > Github & Blog: http://huangjs.github.com/
> >>> > > > > > > >
> >>> > > > > > >
> >>> > > > > >
> >>> > > > > >
> >>> > > > > >
> >>> > > > > > --
> >>> > > > > > Jianshi Huang
> >>> > > > > >
> >>> > > > > > LinkedIn: jianshi
> >>> > > > > > Twitter: @jshuang
> >>> > > > > > Github & Blog: http://huangjs.github.com/
> >>> > > > > >
> >>> > > > >
> >>> > > >
> >>> > > >
> >>> > > >
> >>> > > > --
> >>> > > > Jianshi Huang
> >>> > > >
> >>> > > > LinkedIn: jianshi
> >>> > > > Twitter: @jshuang
> >>> > > > Github & Blog: http://huangjs.github.com/
> >>> > > >
> >>> > >
> >>> >
> >>> >
> >>> >
> >>> > --
> >>> > Jianshi Huang
> >>> >
> >>> > LinkedIn: jianshi
> >>> > Twitter: @jshuang
> >>> > Github & Blog: http://huangjs.github.com/
> >>> >
> >>>
> >>
> >>
> >>
> >> --
> >> Jianshi Huang
> >>
> >> LinkedIn: jianshi
> >> Twitter: @jshuang
> >> Github & Blog: http://huangjs.github.com/
> >>
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Ok, I fixed this by manually reassign region servers to newly created ones.

  def reassignRegionServer(admin: HBaseAdmin, regions: Seq[HRegionInfo],
regionServers: Seq[ServerName]): Unit = {
    val rand = new Random()
    regions.foreach { r =>
      val idx = rand.nextInt(regionServers.size)
      val server = regionServers(idx)
      // FIXME: what if selected region server is dead?
        admin.move(r.getEncodedNameAsBytes,
server.getServerName.getBytes("UTF8"))
    }
  }

er...

Jianshi

On Tue, Sep 23, 2014 at 12:24 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Hmm...any workaround? I only want to do this:
>
> Rebalance the new regions *evenly* to all servers after manually adding
> splits, so later bulk insertions won't cause contention.
>
> P.S.
> Looks like two of the region servers which had majority of the regions
> were down during Major compaction... I guess it had too much data.
>
>
> Jianshi
>
> On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
>> Yes, I have access to Master UI, however logs/*.log cannot be opened or
>> downloaded, must be some security restrictions in the proxy...
>>
>> Jianshi
>>
>> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com> wrote:
>>
>>> Do you have access to Master UI ?
>>>
>>> <master-address>:60010/logs/ would show you list of log files.
>>>
>>> The you can view <master-address>:60010/logs/hbase-<user>-master-XXX.log
>>>
>>> Cheers
>>>
>>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <ji...@gmail.com>
>>> wrote:
>>>
>>> > Ah... I don't have access to HMaster logs... I need to ask the admin.
>>> >
>>> > Jianshi
>>> >
>>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com> wrote:
>>> >
>>> > > bq. assign per-table balancer class
>>> > >
>>> > > No that I know of.
>>> > > Can you pastebin master log involving output from balancer ?
>>> > >
>>> > > Cheers
>>> > >
>>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
>>> jianshi.huang@gmail.com>
>>> > > wrote:
>>> > >
>>> > > > Hi Ted,
>>> > > >
>>> > > > I moved setBalancerRunning before balancer and run them twice.
>>> However
>>> > I
>>> > > > still got highly skewed region distribution.
>>> > > >
>>> > > > I guess it's because of the StochasticLoadBalancer, can I assign
>>> > > per-table
>>> > > > balancer class in HBase?
>>> > > >
>>> > > >
>>> > > > Jianshi
>>> > > >
>>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com>
>>> wrote:
>>> > > >
>>> > > > > admin.setBalancerRunning() call should precede the call to
>>> > > > > admin.balancer().
>>> > > > >
>>> > > > > You can inspect master log to see whether regions are being
>>> moved off
>>> > > the
>>> > > > > heavily loaded server.
>>> > > > >
>>> > > > > Cheers
>>> > > > >
>>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
>>> > > jianshi.huang@gmail.com>
>>> > > > > wrote:
>>> > > > >
>>> > > > > > Hi Ted and others,
>>> > > > > >
>>> > > > > > I did the following after adding splits (without data) to my
>>> table,
>>> > > > > however
>>> > > > > > the region is still very imbalanced (one region server has 221
>>> > > regions
>>> > > > > and
>>> > > > > > other 50 region servers have about 4~8 regions each).
>>> > > > > >
>>> > > > > >       admin.balancer()
>>> > > > > >       admin.setBalancerRunning(true, true)
>>> > > > > >
>>> > > > > > The balancer class in my HBase cluster is
>>> > > > > >
>>> > > > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
>>> > > > > >
>>> > > > > > So, is this behavior expected? Can I assign different balancer
>>> > class
>>> > > to
>>> > > > > my
>>> > > > > > tables (I don't have HBase admin permission)? Which one should
>>> I
>>> > use?
>>> > > > > >
>>> > > > > > I just want HBase to evenly distribute the regions even they
>>> don't
>>> > > have
>>> > > > > > data (that's the purpose of pre-split I think).
>>> > > > > >
>>> > > > > >
>>> > > > > > Jianshi
>>> > > > > >
>>> > > > > >
>>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com>
>>> > wrote:
>>> > > > > >
>>> > > > > > > Yes. See the following method in HBaseAdmin:
>>> > > > > > >
>>> > > > > > >   public boolean balancer()
>>> > > > > > >
>>> > > > > > >
>>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
>>> > > > jianshi.huang@gmail.com
>>> > > > > >
>>> > > > > > > wrote:
>>> > > > > > >
>>> > > > > > > > Thanks Ted!
>>> > > > > > > >
>>> > > > > > > > Didn't know I still need to run the 'balancer' command.
>>> > > > > > > >
>>> > > > > > > > Is there a way to do it programmatically?
>>> > > > > > > >
>>> > > > > > > > Jianshi
>>> > > > > > > >
>>> > > > > > > >
>>> > > > > > > >
>>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
>>> yuzhihong@gmail.com>
>>> > > > wrote:
>>> > > > > > > >
>>> > > > > > > > > After splitting the region, you may need to run balancer
>>> to
>>> > > > spread
>>> > > > > > the
>>> > > > > > > > new
>>> > > > > > > > > regions out.
>>> > > > > > > > >
>>> > > > > > > > > Cheers
>>> > > > > > > > >
>>> > > > > > > > >
>>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
>>> > > > > > jianshi.huang@gmail.com
>>> > > > > > > >
>>> > > > > > > > > wrote:
>>> > > > > > > > >
>>> > > > > > > > > > Hi Shahab,
>>> > > > > > > > > >
>>> > > > > > > > > > I see, that seems to be the right way...
>>> > > > > > > > > >
>>> > > > > > > > > >
>>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
>>> > > > > > > shahab.yunus@gmail.com>
>>> > > > > > > > > > wrote:
>>> > > > > > > > > >
>>> > > > > > > > > > > Shahab
>>> > > > > > > > > >
>>> > > > > > > > > >
>>> > > > > > > > > >
>>> > > > > > > > > >
>>> > > > > > > > > >
>>> > > > > > > > > > --
>>> > > > > > > > > > Jianshi Huang
>>> > > > > > > > > >
>>> > > > > > > > > > LinkedIn: jianshi
>>> > > > > > > > > > Twitter: @jshuang
>>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
>>> > > > > > > > > >
>>> > > > > > > > >
>>> > > > > > > >
>>> > > > > > > >
>>> > > > > > > >
>>> > > > > > > > --
>>> > > > > > > > Jianshi Huang
>>> > > > > > > >
>>> > > > > > > > LinkedIn: jianshi
>>> > > > > > > > Twitter: @jshuang
>>> > > > > > > > Github & Blog: http://huangjs.github.com/
>>> > > > > > > >
>>> > > > > > >
>>> > > > > >
>>> > > > > >
>>> > > > > >
>>> > > > > > --
>>> > > > > > Jianshi Huang
>>> > > > > >
>>> > > > > > LinkedIn: jianshi
>>> > > > > > Twitter: @jshuang
>>> > > > > > Github & Blog: http://huangjs.github.com/
>>> > > > > >
>>> > > > >
>>> > > >
>>> > > >
>>> > > >
>>> > > > --
>>> > > > Jianshi Huang
>>> > > >
>>> > > > LinkedIn: jianshi
>>> > > > Twitter: @jshuang
>>> > > > Github & Blog: http://huangjs.github.com/
>>> > > >
>>> > >
>>> >
>>> >
>>> >
>>> > --
>>> > Jianshi Huang
>>> >
>>> > LinkedIn: jianshi
>>> > Twitter: @jshuang
>>> > Github & Blog: http://huangjs.github.com/
>>> >
>>>
>>
>>
>>
>> --
>> Jianshi Huang
>>
>> LinkedIn: jianshi
>> Twitter: @jshuang
>> Github & Blog: http://huangjs.github.com/
>>
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hmm...any workaround? I only want to do this:

Rebalance the new regions *evenly* to all servers after manually adding
splits, so later bulk insertions won't cause contention.

P.S.
Looks like two of the region servers which had majority of the regions were
down during Major compaction... I guess it had too much data.


Jianshi

On Tue, Sep 23, 2014 at 12:13 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Yes, I have access to Master UI, however logs/*.log cannot be opened or
> downloaded, must be some security restrictions in the proxy...
>
> Jianshi
>
> On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com> wrote:
>
>> Do you have access to Master UI ?
>>
>> <master-address>:60010/logs/ would show you list of log files.
>>
>> The you can view <master-address>:60010/logs/hbase-<user>-master-XXX.log
>>
>> Cheers
>>
>> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <ji...@gmail.com>
>> wrote:
>>
>> > Ah... I don't have access to HMaster logs... I need to ask the admin.
>> >
>> > Jianshi
>> >
>> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com> wrote:
>> >
>> > > bq. assign per-table balancer class
>> > >
>> > > No that I know of.
>> > > Can you pastebin master log involving output from balancer ?
>> > >
>> > > Cheers
>> > >
>> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
>> jianshi.huang@gmail.com>
>> > > wrote:
>> > >
>> > > > Hi Ted,
>> > > >
>> > > > I moved setBalancerRunning before balancer and run them twice.
>> However
>> > I
>> > > > still got highly skewed region distribution.
>> > > >
>> > > > I guess it's because of the StochasticLoadBalancer, can I assign
>> > > per-table
>> > > > balancer class in HBase?
>> > > >
>> > > >
>> > > > Jianshi
>> > > >
>> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com>
>> wrote:
>> > > >
>> > > > > admin.setBalancerRunning() call should precede the call to
>> > > > > admin.balancer().
>> > > > >
>> > > > > You can inspect master log to see whether regions are being moved
>> off
>> > > the
>> > > > > heavily loaded server.
>> > > > >
>> > > > > Cheers
>> > > > >
>> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
>> > > jianshi.huang@gmail.com>
>> > > > > wrote:
>> > > > >
>> > > > > > Hi Ted and others,
>> > > > > >
>> > > > > > I did the following after adding splits (without data) to my
>> table,
>> > > > > however
>> > > > > > the region is still very imbalanced (one region server has 221
>> > > regions
>> > > > > and
>> > > > > > other 50 region servers have about 4~8 regions each).
>> > > > > >
>> > > > > >       admin.balancer()
>> > > > > >       admin.setBalancerRunning(true, true)
>> > > > > >
>> > > > > > The balancer class in my HBase cluster is
>> > > > > >
>> > > > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
>> > > > > >
>> > > > > > So, is this behavior expected? Can I assign different balancer
>> > class
>> > > to
>> > > > > my
>> > > > > > tables (I don't have HBase admin permission)? Which one should I
>> > use?
>> > > > > >
>> > > > > > I just want HBase to evenly distribute the regions even they
>> don't
>> > > have
>> > > > > > data (that's the purpose of pre-split I think).
>> > > > > >
>> > > > > >
>> > > > > > Jianshi
>> > > > > >
>> > > > > >
>> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com>
>> > wrote:
>> > > > > >
>> > > > > > > Yes. See the following method in HBaseAdmin:
>> > > > > > >
>> > > > > > >   public boolean balancer()
>> > > > > > >
>> > > > > > >
>> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
>> > > > jianshi.huang@gmail.com
>> > > > > >
>> > > > > > > wrote:
>> > > > > > >
>> > > > > > > > Thanks Ted!
>> > > > > > > >
>> > > > > > > > Didn't know I still need to run the 'balancer' command.
>> > > > > > > >
>> > > > > > > > Is there a way to do it programmatically?
>> > > > > > > >
>> > > > > > > > Jianshi
>> > > > > > > >
>> > > > > > > >
>> > > > > > > >
>> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <
>> yuzhihong@gmail.com>
>> > > > wrote:
>> > > > > > > >
>> > > > > > > > > After splitting the region, you may need to run balancer
>> to
>> > > > spread
>> > > > > > the
>> > > > > > > > new
>> > > > > > > > > regions out.
>> > > > > > > > >
>> > > > > > > > > Cheers
>> > > > > > > > >
>> > > > > > > > >
>> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
>> > > > > > jianshi.huang@gmail.com
>> > > > > > > >
>> > > > > > > > > wrote:
>> > > > > > > > >
>> > > > > > > > > > Hi Shahab,
>> > > > > > > > > >
>> > > > > > > > > > I see, that seems to be the right way...
>> > > > > > > > > >
>> > > > > > > > > >
>> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
>> > > > > > > shahab.yunus@gmail.com>
>> > > > > > > > > > wrote:
>> > > > > > > > > >
>> > > > > > > > > > > Shahab
>> > > > > > > > > >
>> > > > > > > > > >
>> > > > > > > > > >
>> > > > > > > > > >
>> > > > > > > > > >
>> > > > > > > > > > --
>> > > > > > > > > > Jianshi Huang
>> > > > > > > > > >
>> > > > > > > > > > LinkedIn: jianshi
>> > > > > > > > > > Twitter: @jshuang
>> > > > > > > > > > Github & Blog: http://huangjs.github.com/
>> > > > > > > > > >
>> > > > > > > > >
>> > > > > > > >
>> > > > > > > >
>> > > > > > > >
>> > > > > > > > --
>> > > > > > > > Jianshi Huang
>> > > > > > > >
>> > > > > > > > LinkedIn: jianshi
>> > > > > > > > Twitter: @jshuang
>> > > > > > > > Github & Blog: http://huangjs.github.com/
>> > > > > > > >
>> > > > > > >
>> > > > > >
>> > > > > >
>> > > > > >
>> > > > > > --
>> > > > > > Jianshi Huang
>> > > > > >
>> > > > > > LinkedIn: jianshi
>> > > > > > Twitter: @jshuang
>> > > > > > Github & Blog: http://huangjs.github.com/
>> > > > > >
>> > > > >
>> > > >
>> > > >
>> > > >
>> > > > --
>> > > > Jianshi Huang
>> > > >
>> > > > LinkedIn: jianshi
>> > > > Twitter: @jshuang
>> > > > Github & Blog: http://huangjs.github.com/
>> > > >
>> > >
>> >
>> >
>> >
>> > --
>> > Jianshi Huang
>> >
>> > LinkedIn: jianshi
>> > Twitter: @jshuang
>> > Github & Blog: http://huangjs.github.com/
>> >
>>
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Yes, I have access to Master UI, however logs/*.log cannot be opened or
downloaded, must be some security restrictions in the proxy...

Jianshi

On Tue, Sep 23, 2014 at 12:06 AM, Ted Yu <yu...@gmail.com> wrote:

> Do you have access to Master UI ?
>
> <master-address>:60010/logs/ would show you list of log files.
>
> The you can view <master-address>:60010/logs/hbase-<user>-master-XXX.log
>
> Cheers
>
> On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Ah... I don't have access to HMaster logs... I need to ask the admin.
> >
> > Jianshi
> >
> > On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > bq. assign per-table balancer class
> > >
> > > No that I know of.
> > > Can you pastebin master log involving output from balancer ?
> > >
> > > Cheers
> > >
> > > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <
> jianshi.huang@gmail.com>
> > > wrote:
> > >
> > > > Hi Ted,
> > > >
> > > > I moved setBalancerRunning before balancer and run them twice.
> However
> > I
> > > > still got highly skewed region distribution.
> > > >
> > > > I guess it's because of the StochasticLoadBalancer, can I assign
> > > per-table
> > > > balancer class in HBase?
> > > >
> > > >
> > > > Jianshi
> > > >
> > > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com> wrote:
> > > >
> > > > > admin.setBalancerRunning() call should precede the call to
> > > > > admin.balancer().
> > > > >
> > > > > You can inspect master log to see whether regions are being moved
> off
> > > the
> > > > > heavily loaded server.
> > > > >
> > > > > Cheers
> > > > >
> > > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com>
> > > > > wrote:
> > > > >
> > > > > > Hi Ted and others,
> > > > > >
> > > > > > I did the following after adding splits (without data) to my
> table,
> > > > > however
> > > > > > the region is still very imbalanced (one region server has 221
> > > regions
> > > > > and
> > > > > > other 50 region servers have about 4~8 regions each).
> > > > > >
> > > > > >       admin.balancer()
> > > > > >       admin.setBalancerRunning(true, true)
> > > > > >
> > > > > > The balancer class in my HBase cluster is
> > > > > >
> > > > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > > > >
> > > > > > So, is this behavior expected? Can I assign different balancer
> > class
> > > to
> > > > > my
> > > > > > tables (I don't have HBase admin permission)? Which one should I
> > use?
> > > > > >
> > > > > > I just want HBase to evenly distribute the regions even they
> don't
> > > have
> > > > > > data (that's the purpose of pre-split I think).
> > > > > >
> > > > > >
> > > > > > Jianshi
> > > > > >
> > > > > >
> > > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com>
> > wrote:
> > > > > >
> > > > > > > Yes. See the following method in HBaseAdmin:
> > > > > > >
> > > > > > >   public boolean balancer()
> > > > > > >
> > > > > > >
> > > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > > > jianshi.huang@gmail.com
> > > > > >
> > > > > > > wrote:
> > > > > > >
> > > > > > > > Thanks Ted!
> > > > > > > >
> > > > > > > > Didn't know I still need to run the 'balancer' command.
> > > > > > > >
> > > > > > > > Is there a way to do it programmatically?
> > > > > > > >
> > > > > > > > Jianshi
> > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yuzhihong@gmail.com
> >
> > > > wrote:
> > > > > > > >
> > > > > > > > > After splitting the region, you may need to run balancer to
> > > > spread
> > > > > > the
> > > > > > > > new
> > > > > > > > > regions out.
> > > > > > > > >
> > > > > > > > > Cheers
> > > > > > > > >
> > > > > > > > >
> > > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > > > > jianshi.huang@gmail.com
> > > > > > > >
> > > > > > > > > wrote:
> > > > > > > > >
> > > > > > > > > > Hi Shahab,
> > > > > > > > > >
> > > > > > > > > > I see, that seems to be the right way...
> > > > > > > > > >
> > > > > > > > > >
> > > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > > > > shahab.yunus@gmail.com>
> > > > > > > > > > wrote:
> > > > > > > > > >
> > > > > > > > > > > Shahab
> > > > > > > > > >
> > > > > > > > > >
> > > > > > > > > >
> > > > > > > > > >
> > > > > > > > > >
> > > > > > > > > > --
> > > > > > > > > > Jianshi Huang
> > > > > > > > > >
> > > > > > > > > > LinkedIn: jianshi
> > > > > > > > > > Twitter: @jshuang
> > > > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > > > > >
> > > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > > --
> > > > > > > > Jianshi Huang
> > > > > > > >
> > > > > > > > LinkedIn: jianshi
> > > > > > > > Twitter: @jshuang
> > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > > >
> > > > > > >
> > > > > >
> > > > > >
> > > > > >
> > > > > > --
> > > > > > Jianshi Huang
> > > > > >
> > > > > > LinkedIn: jianshi
> > > > > > Twitter: @jshuang
> > > > > > Github & Blog: http://huangjs.github.com/
> > > > > >
> > > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
Do you have access to Master UI ?

<master-address>:60010/logs/ would show you list of log files.

The you can view <master-address>:60010/logs/hbase-<user>-master-XXX.log

Cheers

On Mon, Sep 22, 2014 at 9:00 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Ah... I don't have access to HMaster logs... I need to ask the admin.
>
> Jianshi
>
> On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com> wrote:
>
> > bq. assign per-table balancer class
> >
> > No that I know of.
> > Can you pastebin master log involving output from balancer ?
> >
> > Cheers
> >
> > On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <ji...@gmail.com>
> > wrote:
> >
> > > Hi Ted,
> > >
> > > I moved setBalancerRunning before balancer and run them twice. However
> I
> > > still got highly skewed region distribution.
> > >
> > > I guess it's because of the StochasticLoadBalancer, can I assign
> > per-table
> > > balancer class in HBase?
> > >
> > >
> > > Jianshi
> > >
> > > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com> wrote:
> > >
> > > > admin.setBalancerRunning() call should precede the call to
> > > > admin.balancer().
> > > >
> > > > You can inspect master log to see whether regions are being moved off
> > the
> > > > heavily loaded server.
> > > >
> > > > Cheers
> > > >
> > > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> > jianshi.huang@gmail.com>
> > > > wrote:
> > > >
> > > > > Hi Ted and others,
> > > > >
> > > > > I did the following after adding splits (without data) to my table,
> > > > however
> > > > > the region is still very imbalanced (one region server has 221
> > regions
> > > > and
> > > > > other 50 region servers have about 4~8 regions each).
> > > > >
> > > > >       admin.balancer()
> > > > >       admin.setBalancerRunning(true, true)
> > > > >
> > > > > The balancer class in my HBase cluster is
> > > > >
> > > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > > >
> > > > > So, is this behavior expected? Can I assign different balancer
> class
> > to
> > > > my
> > > > > tables (I don't have HBase admin permission)? Which one should I
> use?
> > > > >
> > > > > I just want HBase to evenly distribute the regions even they don't
> > have
> > > > > data (that's the purpose of pre-split I think).
> > > > >
> > > > >
> > > > > Jianshi
> > > > >
> > > > >
> > > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com>
> wrote:
> > > > >
> > > > > > Yes. See the following method in HBaseAdmin:
> > > > > >
> > > > > >   public boolean balancer()
> > > > > >
> > > > > >
> > > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com
> > > > >
> > > > > > wrote:
> > > > > >
> > > > > > > Thanks Ted!
> > > > > > >
> > > > > > > Didn't know I still need to run the 'balancer' command.
> > > > > > >
> > > > > > > Is there a way to do it programmatically?
> > > > > > >
> > > > > > > Jianshi
> > > > > > >
> > > > > > >
> > > > > > >
> > > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com>
> > > wrote:
> > > > > > >
> > > > > > > > After splitting the region, you may need to run balancer to
> > > spread
> > > > > the
> > > > > > > new
> > > > > > > > regions out.
> > > > > > > >
> > > > > > > > Cheers
> > > > > > > >
> > > > > > > >
> > > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > > > jianshi.huang@gmail.com
> > > > > > >
> > > > > > > > wrote:
> > > > > > > >
> > > > > > > > > Hi Shahab,
> > > > > > > > >
> > > > > > > > > I see, that seems to be the right way...
> > > > > > > > >
> > > > > > > > >
> > > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > > > shahab.yunus@gmail.com>
> > > > > > > > > wrote:
> > > > > > > > >
> > > > > > > > > > Shahab
> > > > > > > > >
> > > > > > > > >
> > > > > > > > >
> > > > > > > > >
> > > > > > > > >
> > > > > > > > > --
> > > > > > > > > Jianshi Huang
> > > > > > > > >
> > > > > > > > > LinkedIn: jianshi
> > > > > > > > > Twitter: @jshuang
> > > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > > > >
> > > > > > > >
> > > > > > >
> > > > > > >
> > > > > > >
> > > > > > > --
> > > > > > > Jianshi Huang
> > > > > > >
> > > > > > > LinkedIn: jianshi
> > > > > > > Twitter: @jshuang
> > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > >
> > > > > >
> > > > >
> > > > >
> > > > >
> > > > > --
> > > > > Jianshi Huang
> > > > >
> > > > > LinkedIn: jianshi
> > > > > Twitter: @jshuang
> > > > > Github & Blog: http://huangjs.github.com/
> > > > >
> > > >
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Ah... I don't have access to HMaster logs... I need to ask the admin.

Jianshi

On Mon, Sep 22, 2014 at 11:49 PM, Ted Yu <yu...@gmail.com> wrote:

> bq. assign per-table balancer class
>
> No that I know of.
> Can you pastebin master log involving output from balancer ?
>
> Cheers
>
> On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Hi Ted,
> >
> > I moved setBalancerRunning before balancer and run them twice. However I
> > still got highly skewed region distribution.
> >
> > I guess it's because of the StochasticLoadBalancer, can I assign
> per-table
> > balancer class in HBase?
> >
> >
> > Jianshi
> >
> > On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > admin.setBalancerRunning() call should precede the call to
> > > admin.balancer().
> > >
> > > You can inspect master log to see whether regions are being moved off
> the
> > > heavily loaded server.
> > >
> > > Cheers
> > >
> > > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <
> jianshi.huang@gmail.com>
> > > wrote:
> > >
> > > > Hi Ted and others,
> > > >
> > > > I did the following after adding splits (without data) to my table,
> > > however
> > > > the region is still very imbalanced (one region server has 221
> regions
> > > and
> > > > other 50 region servers have about 4~8 regions each).
> > > >
> > > >       admin.balancer()
> > > >       admin.setBalancerRunning(true, true)
> > > >
> > > > The balancer class in my HBase cluster is
> > > >
> > > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > > >
> > > > So, is this behavior expected? Can I assign different balancer class
> to
> > > my
> > > > tables (I don't have HBase admin permission)? Which one should I use?
> > > >
> > > > I just want HBase to evenly distribute the regions even they don't
> have
> > > > data (that's the purpose of pre-split I think).
> > > >
> > > >
> > > > Jianshi
> > > >
> > > >
> > > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com> wrote:
> > > >
> > > > > Yes. See the following method in HBaseAdmin:
> > > > >
> > > > >   public boolean balancer()
> > > > >
> > > > >
> > > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> > jianshi.huang@gmail.com
> > > >
> > > > > wrote:
> > > > >
> > > > > > Thanks Ted!
> > > > > >
> > > > > > Didn't know I still need to run the 'balancer' command.
> > > > > >
> > > > > > Is there a way to do it programmatically?
> > > > > >
> > > > > > Jianshi
> > > > > >
> > > > > >
> > > > > >
> > > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com>
> > wrote:
> > > > > >
> > > > > > > After splitting the region, you may need to run balancer to
> > spread
> > > > the
> > > > > > new
> > > > > > > regions out.
> > > > > > >
> > > > > > > Cheers
> > > > > > >
> > > > > > >
> > > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > > jianshi.huang@gmail.com
> > > > > >
> > > > > > > wrote:
> > > > > > >
> > > > > > > > Hi Shahab,
> > > > > > > >
> > > > > > > > I see, that seems to be the right way...
> > > > > > > >
> > > > > > > >
> > > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > > shahab.yunus@gmail.com>
> > > > > > > > wrote:
> > > > > > > >
> > > > > > > > > Shahab
> > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > >
> > > > > > > > --
> > > > > > > > Jianshi Huang
> > > > > > > >
> > > > > > > > LinkedIn: jianshi
> > > > > > > > Twitter: @jshuang
> > > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > > >
> > > > > > >
> > > > > >
> > > > > >
> > > > > >
> > > > > > --
> > > > > > Jianshi Huang
> > > > > >
> > > > > > LinkedIn: jianshi
> > > > > > Twitter: @jshuang
> > > > > > Github & Blog: http://huangjs.github.com/
> > > > > >
> > > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
bq. assign per-table balancer class

No that I know of.
Can you pastebin master log involving output from balancer ?

Cheers

On Mon, Sep 22, 2014 at 8:29 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Hi Ted,
>
> I moved setBalancerRunning before balancer and run them twice. However I
> still got highly skewed region distribution.
>
> I guess it's because of the StochasticLoadBalancer, can I assign per-table
> balancer class in HBase?
>
>
> Jianshi
>
> On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com> wrote:
>
> > admin.setBalancerRunning() call should precede the call to
> > admin.balancer().
> >
> > You can inspect master log to see whether regions are being moved off the
> > heavily loaded server.
> >
> > Cheers
> >
> > On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <ji...@gmail.com>
> > wrote:
> >
> > > Hi Ted and others,
> > >
> > > I did the following after adding splits (without data) to my table,
> > however
> > > the region is still very imbalanced (one region server has 221 regions
> > and
> > > other 50 region servers have about 4~8 regions each).
> > >
> > >       admin.balancer()
> > >       admin.setBalancerRunning(true, true)
> > >
> > > The balancer class in my HBase cluster is
> > >
> > > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> > >
> > > So, is this behavior expected? Can I assign different balancer class to
> > my
> > > tables (I don't have HBase admin permission)? Which one should I use?
> > >
> > > I just want HBase to evenly distribute the regions even they don't have
> > > data (that's the purpose of pre-split I think).
> > >
> > >
> > > Jianshi
> > >
> > >
> > > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com> wrote:
> > >
> > > > Yes. See the following method in HBaseAdmin:
> > > >
> > > >   public boolean balancer()
> > > >
> > > >
> > > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > > wrote:
> > > >
> > > > > Thanks Ted!
> > > > >
> > > > > Didn't know I still need to run the 'balancer' command.
> > > > >
> > > > > Is there a way to do it programmatically?
> > > > >
> > > > > Jianshi
> > > > >
> > > > >
> > > > >
> > > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com>
> wrote:
> > > > >
> > > > > > After splitting the region, you may need to run balancer to
> spread
> > > the
> > > > > new
> > > > > > regions out.
> > > > > >
> > > > > > Cheers
> > > > > >
> > > > > >
> > > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > > jianshi.huang@gmail.com
> > > > >
> > > > > > wrote:
> > > > > >
> > > > > > > Hi Shahab,
> > > > > > >
> > > > > > > I see, that seems to be the right way...
> > > > > > >
> > > > > > >
> > > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > > shahab.yunus@gmail.com>
> > > > > > > wrote:
> > > > > > >
> > > > > > > > Shahab
> > > > > > >
> > > > > > >
> > > > > > >
> > > > > > >
> > > > > > >
> > > > > > > --
> > > > > > > Jianshi Huang
> > > > > > >
> > > > > > > LinkedIn: jianshi
> > > > > > > Twitter: @jshuang
> > > > > > > Github & Blog: http://huangjs.github.com/
> > > > > > >
> > > > > >
> > > > >
> > > > >
> > > > >
> > > > > --
> > > > > Jianshi Huang
> > > > >
> > > > > LinkedIn: jianshi
> > > > > Twitter: @jshuang
> > > > > Github & Blog: http://huangjs.github.com/
> > > > >
> > > >
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hi Ted,

I moved setBalancerRunning before balancer and run them twice. However I
still got highly skewed region distribution.

I guess it's because of the StochasticLoadBalancer, can I assign per-table
balancer class in HBase?


Jianshi

On Mon, Sep 22, 2014 at 9:50 PM, Ted Yu <yu...@gmail.com> wrote:

> admin.setBalancerRunning() call should precede the call to
> admin.balancer().
>
> You can inspect master log to see whether regions are being moved off the
> heavily loaded server.
>
> Cheers
>
> On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Hi Ted and others,
> >
> > I did the following after adding splits (without data) to my table,
> however
> > the region is still very imbalanced (one region server has 221 regions
> and
> > other 50 region servers have about 4~8 regions each).
> >
> >       admin.balancer()
> >       admin.setBalancerRunning(true, true)
> >
> > The balancer class in my HBase cluster is
> >
> > org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
> >
> > So, is this behavior expected? Can I assign different balancer class to
> my
> > tables (I don't have HBase admin permission)? Which one should I use?
> >
> > I just want HBase to evenly distribute the regions even they don't have
> > data (that's the purpose of pre-split I think).
> >
> >
> > Jianshi
> >
> >
> > On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > Yes. See the following method in HBaseAdmin:
> > >
> > >   public boolean balancer()
> > >
> > >
> > > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <jianshi.huang@gmail.com
> >
> > > wrote:
> > >
> > > > Thanks Ted!
> > > >
> > > > Didn't know I still need to run the 'balancer' command.
> > > >
> > > > Is there a way to do it programmatically?
> > > >
> > > > Jianshi
> > > >
> > > >
> > > >
> > > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com> wrote:
> > > >
> > > > > After splitting the region, you may need to run balancer to spread
> > the
> > > > new
> > > > > regions out.
> > > > >
> > > > > Cheers
> > > > >
> > > > >
> > > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> > jianshi.huang@gmail.com
> > > >
> > > > > wrote:
> > > > >
> > > > > > Hi Shahab,
> > > > > >
> > > > > > I see, that seems to be the right way...
> > > > > >
> > > > > >
> > > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > > shahab.yunus@gmail.com>
> > > > > > wrote:
> > > > > >
> > > > > > > Shahab
> > > > > >
> > > > > >
> > > > > >
> > > > > >
> > > > > >
> > > > > > --
> > > > > > Jianshi Huang
> > > > > >
> > > > > > LinkedIn: jianshi
> > > > > > Twitter: @jshuang
> > > > > > Github & Blog: http://huangjs.github.com/
> > > > > >
> > > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
admin.setBalancerRunning() call should precede the call to admin.balancer().

You can inspect master log to see whether regions are being moved off the
heavily loaded server.

Cheers

On Mon, Sep 22, 2014 at 1:42 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Hi Ted and others,
>
> I did the following after adding splits (without data) to my table, however
> the region is still very imbalanced (one region server has 221 regions and
> other 50 region servers have about 4~8 regions each).
>
>       admin.balancer()
>       admin.setBalancerRunning(true, true)
>
> The balancer class in my HBase cluster is
>
> org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer
>
> So, is this behavior expected? Can I assign different balancer class to my
> tables (I don't have HBase admin permission)? Which one should I use?
>
> I just want HBase to evenly distribute the regions even they don't have
> data (that's the purpose of pre-split I think).
>
>
> Jianshi
>
>
> On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com> wrote:
>
> > Yes. See the following method in HBaseAdmin:
> >
> >   public boolean balancer()
> >
> >
> > On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <ji...@gmail.com>
> > wrote:
> >
> > > Thanks Ted!
> > >
> > > Didn't know I still need to run the 'balancer' command.
> > >
> > > Is there a way to do it programmatically?
> > >
> > > Jianshi
> > >
> > >
> > >
> > > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com> wrote:
> > >
> > > > After splitting the region, you may need to run balancer to spread
> the
> > > new
> > > > regions out.
> > > >
> > > > Cheers
> > > >
> > > >
> > > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <
> jianshi.huang@gmail.com
> > >
> > > > wrote:
> > > >
> > > > > Hi Shahab,
> > > > >
> > > > > I see, that seems to be the right way...
> > > > >
> > > > >
> > > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> > shahab.yunus@gmail.com>
> > > > > wrote:
> > > > >
> > > > > > Shahab
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > > --
> > > > > Jianshi Huang
> > > > >
> > > > > LinkedIn: jianshi
> > > > > Twitter: @jshuang
> > > > > Github & Blog: http://huangjs.github.com/
> > > > >
> > > >
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hi Ted and others,

I did the following after adding splits (without data) to my table, however
the region is still very imbalanced (one region server has 221 regions and
other 50 region servers have about 4~8 regions each).

      admin.balancer()
      admin.setBalancerRunning(true, true)

The balancer class in my HBase cluster is

org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer

So, is this behavior expected? Can I assign different balancer class to my
tables (I don't have HBase admin permission)? Which one should I use?

I just want HBase to evenly distribute the regions even they don't have
data (that's the purpose of pre-split I think).


Jianshi


On Sat, Sep 6, 2014 at 12:45 AM, Ted Yu <yu...@gmail.com> wrote:

> Yes. See the following method in HBaseAdmin:
>
>   public boolean balancer()
>
>
> On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Thanks Ted!
> >
> > Didn't know I still need to run the 'balancer' command.
> >
> > Is there a way to do it programmatically?
> >
> > Jianshi
> >
> >
> >
> > On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com> wrote:
> >
> > > After splitting the region, you may need to run balancer to spread the
> > new
> > > regions out.
> > >
> > > Cheers
> > >
> > >
> > > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <jianshi.huang@gmail.com
> >
> > > wrote:
> > >
> > > > Hi Shahab,
> > > >
> > > > I see, that seems to be the right way...
> > > >
> > > >
> > > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <
> shahab.yunus@gmail.com>
> > > > wrote:
> > > >
> > > > > Shahab
> > > >
> > > >
> > > >
> > > >
> > > >
> > > > --
> > > > Jianshi Huang
> > > >
> > > > LinkedIn: jianshi
> > > > Twitter: @jshuang
> > > > Github & Blog: http://huangjs.github.com/
> > > >
> > >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
Yes. See the following method in HBaseAdmin:

  public boolean balancer()


On Fri, Sep 5, 2014 at 9:38 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Thanks Ted!
>
> Didn't know I still need to run the 'balancer' command.
>
> Is there a way to do it programmatically?
>
> Jianshi
>
>
>
> On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com> wrote:
>
> > After splitting the region, you may need to run balancer to spread the
> new
> > regions out.
> >
> > Cheers
> >
> >
> > On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <ji...@gmail.com>
> > wrote:
> >
> > > Hi Shahab,
> > >
> > > I see, that seems to be the right way...
> > >
> > >
> > > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <sh...@gmail.com>
> > > wrote:
> > >
> > > > Shahab
> > >
> > >
> > >
> > >
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Thanks Ted!

Didn't know I still need to run the 'balancer' command.

Is there a way to do it programmatically?

Jianshi



On Sat, Sep 6, 2014 at 12:29 AM, Ted Yu <yu...@gmail.com> wrote:

> After splitting the region, you may need to run balancer to spread the new
> regions out.
>
> Cheers
>
>
> On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <ji...@gmail.com>
> wrote:
>
> > Hi Shahab,
> >
> > I see, that seems to be the right way...
> >
> >
> > On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <sh...@gmail.com>
> > wrote:
> >
> > > Shahab
> >
> >
> >
> >
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Ted Yu <yu...@gmail.com>.
After splitting the region, you may need to run balancer to spread the new
regions out.

Cheers


On Fri, Sep 5, 2014 at 9:25 AM, Jianshi Huang <ji...@gmail.com>
wrote:

> Hi Shahab,
>
> I see, that seems to be the right way...
>
>
> On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <sh...@gmail.com>
> wrote:
>
> > Shahab
>
>
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hi Shahab,

I see, that seems to be the right way...


On Sat, Sep 6, 2014 at 12:21 AM, Shahab Yunus <sh...@gmail.com>
wrote:

> Shahab





-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Shahab Yunus <sh...@gmail.com>.
For monotonically increasing data: can you try to do pre-splitting of the
destination table? That can help in avoiding one region getting overloaded
at the time of bulkimport.

Regards,
Shahab


On Fri, Sep 5, 2014 at 12:14 PM, Jianshi Huang <ji...@gmail.com>
wrote:

> Thanks Ted, I'll try to do a major compact.
>
> Hi Steven,
>
> Yes, most of my rows are hashed to make it randomly distributed, but one
> column family has monotonically increasing rowkeys, and it's used for
> recording sequence of events.
>
> Do you have a solution how to bulk import this kind of data?
>
> Jianshi
>
>
>
> On Sat, Sep 6, 2014 at 12:00 AM, Magana-zook, Steven Alan <
> maganazook1@llnl.gov> wrote:
>
> > Hi Jianshi,
> >
> > What are the field(s) in your row key? If your row key is monotonically
> > increasing then you will be sending all of your requests to one region
> > server. Even after the region splits, all new entries will keep punishing
> > one server (the region responsible for the split containing the new
> keys).
> >
> > See these articles that may help if this is indeed your issue:
> > 1. http://hbase.apache.org/book/rowkey.design.html
> > 2.
> >
> http://ikaisays.com/2011/01/25/app-engine-datastore-tip-monotonically-incre
> > asing-values-are-bad/
> >
> > Regards,
> > Steven Magana-Zook
> >
> >
> >
> >
> >
> >
> > On 9/5/14 8:54 AM, "Jianshi Huang" <ji...@gmail.com> wrote:
> >
> > >Hi JM,
> > >
> > >What do you mean by the 'destination cluster'? The files are in the same
> > >Hadoop/HDFS cluster where HBase is running.
> > >
> > >Do you mean do the bulk importing on HBase Master node?
> > >
> > >
> > >Jianshi
> > >
> > >
> > >On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
> > >jean-marc@spaggiari.org> wrote:
> > >
> > >> Hi Jianshi,
> > >>
> > >> You might want to upload the file on the destination cluster first and
> > >>then
> > >> re-run your bulk load from there. That way the transfer time will not
> be
> > >> taken into consideration for the timeout size the files will be local.
> > >>
> > >> JM
> > >>
> > >>
> > >> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
> > >>
> > >> > I'm importing 2TB of generated HFiles to HBase and I constantly get
> > >>the
> > >> > following errors:
> > >> >
> > >> > Caused by:
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.h
> > >>base.RegionTooBusyException):
> > >> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock
> > >>in
> > >> > 60000 ms.
> > >> >
> > >> >
> > >>
> >
> >>regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f
> > >>3565642d96.,
> > >> > server=xxxxx.xxx.xxx,60020,1404854700728
> > >> >         at
> > >> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
> > >> >         at
> > >> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRe
> > >>gion.java:5795)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
> > >>3543)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
> > >>3525)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionS
> > >>erver.java:3277)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.c
> > >>allBlockingMethod(ClientProtos.java:28863)
> > >> >         at
> > >> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
> > >> >         at
> > >>org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcSche
> > >>duler.java:160)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcSchedu
> > >>ler.java:38)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.j
> > >>ava:110)
> > >> >         at java.lang.Thread.run(Thread.java:724)
> > >> >
> > >> >         at
> > >> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1
> > >>684)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.ca
> > >>llBlockingMethod(RpcClient.java:1737)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$Blo
> > >>ckingStub.bulkLoadHFile(ClientProtos.java:29276)
> > >> >         at
> > >> >
> > >> >
> > >>
> >
> >>org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.
> > >>java:1548)
> > >> >         ... 11 more
> > >> >
> > >> >
> > >> > What makes the region too busy? Is there a way to improve it?
> > >> >
> > >> > Does that also mean some part of my data are not correctly imported?
> > >> >
> > >> >
> > >> > Thanks,
> > >> >
> > >> > --
> > >> > Jianshi Huang
> > >> >
> > >> > LinkedIn: jianshi
> > >> > Twitter: @jshuang
> > >> > Github & Blog: http://huangjs.github.com/
> > >> >
> > >>
> > >
> > >
> > >
> > >--
> > >Jianshi Huang
> > >
> > >LinkedIn: jianshi
> > >Twitter: @jshuang
> > >Github & Blog: http://huangjs.github.com/
> >
> >
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hi Steven,

I did 1) and 2) and the error was during LoadIncrementalHFiles.

I can't do 3) because that CF is mostly used for mapreduce inputs, so a
continuous rowkey is preferred.

Jianshi



On Sat, Sep 6, 2014 at 12:29 AM, Magana-zook, Steven Alan <
maganazook1@llnl.gov> wrote:

> Jianshi,
>
> I have seen many solutions to importing this kind of data:
>
> 1. Pre-splitting regions (I did not try this)
>
> 2. Using a map reduce job to create HFiles instead of putting individual
> rows into the database
> (instructions here: http://hbase.apache.org/book/arch.bulk.load.html
>
> 3. Modifying the row key to not be monotonic
>
> I went with the third solution by pre-prending a random integer before the
> other fields in my composite row key ( "<random int>_<key field 1>_<key
> field 2>Š.")
>
> When you make any changes, you can verify it is working by viewing the
> Hbase web interface (port 60010 on the hbase master) to see the requests
> per second on the various region servers.
>
>
> Thank you,
> Steven Magana-Zook
>
>
>
>
>
>
> On 9/5/14 9:14 AM, "Jianshi Huang" <ji...@gmail.com> wrote:
>
> >Thanks Ted, I'll try to do a major compact.
> >
> >Hi Steven,
> >
> >Yes, most of my rows are hashed to make it randomly distributed, but one
> >column family has monotonically increasing rowkeys, and it's used for
> >recording sequence of events.
> >
> >Do you have a solution how to bulk import this kind of data?
> >
> >Jianshi
> >
> >
> >
> >On Sat, Sep 6, 2014 at 12:00 AM, Magana-zook, Steven Alan <
> >maganazook1@llnl.gov> wrote:
> >
> >> Hi Jianshi,
> >>
> >> What are the field(s) in your row key? If your row key is monotonically
> >> increasing then you will be sending all of your requests to one region
> >> server. Even after the region splits, all new entries will keep
> >>punishing
> >> one server (the region responsible for the split containing the new
> >>keys).
> >>
> >> See these articles that may help if this is indeed your issue:
> >> 1. http://hbase.apache.org/book/rowkey.design.html
> >> 2.
> >>
> >>
> http://ikaisays.com/2011/01/25/app-engine-datastore-tip-monotonically-inc
> >>re
> >> asing-values-are-bad/
> >>
> >> Regards,
> >> Steven Magana-Zook
> >>
> >>
> >>
> >>
> >>
> >>
> >> On 9/5/14 8:54 AM, "Jianshi Huang" <ji...@gmail.com> wrote:
> >>
> >> >Hi JM,
> >> >
> >> >What do you mean by the 'destination cluster'? The files are in the
> >>same
> >> >Hadoop/HDFS cluster where HBase is running.
> >> >
> >> >Do you mean do the bulk importing on HBase Master node?
> >> >
> >> >
> >> >Jianshi
> >> >
> >> >
> >> >On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
> >> >jean-marc@spaggiari.org> wrote:
> >> >
> >> >> Hi Jianshi,
> >> >>
> >> >> You might want to upload the file on the destination cluster first
> >>and
> >> >>then
> >> >> re-run your bulk load from there. That way the transfer time will
> >>not be
> >> >> taken into consideration for the timeout size the files will be
> >>local.
> >> >>
> >> >> JM
> >> >>
> >> >>
> >> >> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
> >> >>
> >> >> > I'm importing 2TB of generated HFiles to HBase and I constantly get
> >> >>the
> >> >> > following errors:
> >> >> >
> >> >> > Caused by:
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop
> >>>>.h
> >> >>base.RegionTooBusyException):
> >> >> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a
> >>lock
> >> >>in
> >> >> > 60000 ms.
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de73
> >>>>3f
> >> >>3565642d96.,
> >> >> > server=xxxxx.xxx.xxx,60020,1404854700728
> >> >> >         at
> >> >> >
> >>org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
> >> >> >         at
> >> >> >
> >>org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(H
> >>>>Re
> >> >>gion.java:5795)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.jav
> >>>>a:
> >> >>3543)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.jav
> >>>>a:
> >> >>3525)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegio
> >>>>nS
> >> >>erver.java:3277)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2
> >>>>.c
> >> >>allBlockingMethod(ClientProtos.java:28863)
> >> >> >         at
> >> >> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
> >> >> >         at
> >> >>org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcSc
> >>>>he
> >> >>duler.java:160)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcSche
> >>>>du
> >> >>ler.java:38)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler
> >>>>.j
> >> >>ava:110)
> >> >> >         at java.lang.Thread.run(Thread.java:724)
> >> >> >
> >> >> >         at
> >> >> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java
> >>>>:1
> >> >>684)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.
> >>>>ca
> >> >>llBlockingMethod(RpcClient.java:1737)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$B
> >>>>lo
> >> >>ckingStub.bulkLoadHFile(ClientProtos.java:29276)
> >> >> >         at
> >> >> >
> >> >> >
> >> >>
> >>
> >>>>org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUti
> >>>>l.
> >> >>java:1548)
> >> >> >         ... 11 more
> >> >> >
> >> >> >
> >> >> > What makes the region too busy? Is there a way to improve it?
> >> >> >
> >> >> > Does that also mean some part of my data are not correctly
> >>imported?
> >> >> >
> >> >> >
> >> >> > Thanks,
> >> >> >
> >> >> > --
> >> >> > Jianshi Huang
> >> >> >
> >> >> > LinkedIn: jianshi
> >> >> > Twitter: @jshuang
> >> >> > Github & Blog: http://huangjs.github.com/
> >> >> >
> >> >>
> >> >
> >> >
> >> >
> >> >--
> >> >Jianshi Huang
> >> >
> >> >LinkedIn: jianshi
> >> >Twitter: @jshuang
> >> >Github & Blog: http://huangjs.github.com/
> >>
> >>
> >
> >
> >--
> >Jianshi Huang
> >
> >LinkedIn: jianshi
> >Twitter: @jshuang
> >Github & Blog: http://huangjs.github.com/
>
>


-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by "Magana-zook, Steven Alan" <ma...@llnl.gov>.
Jianshi,

I have seen many solutions to importing this kind of data:

1. Pre-splitting regions (I did not try this)

2. Using a map reduce job to create HFiles instead of putting individual
rows into the database
(instructions here: http://hbase.apache.org/book/arch.bulk.load.html

3. Modifying the row key to not be monotonic

I went with the third solution by pre-prending a random integer before the
other fields in my composite row key ( "<random int>_<key field 1>_<key
field 2>Š.")

When you make any changes, you can verify it is working by viewing the
Hbase web interface (port 60010 on the hbase master) to see the requests
per second on the various region servers.


Thank you,
Steven Magana-Zook






On 9/5/14 9:14 AM, "Jianshi Huang" <ji...@gmail.com> wrote:

>Thanks Ted, I'll try to do a major compact.
>
>Hi Steven,
>
>Yes, most of my rows are hashed to make it randomly distributed, but one
>column family has monotonically increasing rowkeys, and it's used for
>recording sequence of events.
>
>Do you have a solution how to bulk import this kind of data?
>
>Jianshi
>
>
>
>On Sat, Sep 6, 2014 at 12:00 AM, Magana-zook, Steven Alan <
>maganazook1@llnl.gov> wrote:
>
>> Hi Jianshi,
>>
>> What are the field(s) in your row key? If your row key is monotonically
>> increasing then you will be sending all of your requests to one region
>> server. Even after the region splits, all new entries will keep
>>punishing
>> one server (the region responsible for the split containing the new
>>keys).
>>
>> See these articles that may help if this is indeed your issue:
>> 1. http://hbase.apache.org/book/rowkey.design.html
>> 2.
>> 
>>http://ikaisays.com/2011/01/25/app-engine-datastore-tip-monotonically-inc
>>re
>> asing-values-are-bad/
>>
>> Regards,
>> Steven Magana-Zook
>>
>>
>>
>>
>>
>>
>> On 9/5/14 8:54 AM, "Jianshi Huang" <ji...@gmail.com> wrote:
>>
>> >Hi JM,
>> >
>> >What do you mean by the 'destination cluster'? The files are in the
>>same
>> >Hadoop/HDFS cluster where HBase is running.
>> >
>> >Do you mean do the bulk importing on HBase Master node?
>> >
>> >
>> >Jianshi
>> >
>> >
>> >On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
>> >jean-marc@spaggiari.org> wrote:
>> >
>> >> Hi Jianshi,
>> >>
>> >> You might want to upload the file on the destination cluster first
>>and
>> >>then
>> >> re-run your bulk load from there. That way the transfer time will
>>not be
>> >> taken into consideration for the timeout size the files will be
>>local.
>> >>
>> >> JM
>> >>
>> >>
>> >> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
>> >>
>> >> > I'm importing 2TB of generated HFiles to HBase and I constantly get
>> >>the
>> >> > following errors:
>> >> >
>> >> > Caused by:
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop
>>>>.h
>> >>base.RegionTooBusyException):
>> >> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a
>>lock
>> >>in
>> >> > 60000 ms.
>> >> >
>> >> >
>> >>
>> 
>>>>regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de73
>>>>3f
>> >>3565642d96.,
>> >> > server=xxxxx.xxx.xxx,60020,1404854700728
>> >> >         at
>> >> > 
>>org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
>> >> >         at
>> >> > 
>>org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(H
>>>>Re
>> >>gion.java:5795)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.jav
>>>>a:
>> >>3543)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.jav
>>>>a:
>> >>3525)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegio
>>>>nS
>> >>erver.java:3277)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2
>>>>.c
>> >>allBlockingMethod(ClientProtos.java:28863)
>> >> >         at
>> >> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
>> >> >         at
>> >>org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcSc
>>>>he
>> >>duler.java:160)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcSche
>>>>du
>> >>ler.java:38)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler
>>>>.j
>> >>ava:110)
>> >> >         at java.lang.Thread.run(Thread.java:724)
>> >> >
>> >> >         at
>> >> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java
>>>>:1
>> >>684)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.
>>>>ca
>> >>llBlockingMethod(RpcClient.java:1737)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$B
>>>>lo
>> >>ckingStub.bulkLoadHFile(ClientProtos.java:29276)
>> >> >         at
>> >> >
>> >> >
>> >>
>> 
>>>>org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUti
>>>>l.
>> >>java:1548)
>> >> >         ... 11 more
>> >> >
>> >> >
>> >> > What makes the region too busy? Is there a way to improve it?
>> >> >
>> >> > Does that also mean some part of my data are not correctly
>>imported?
>> >> >
>> >> >
>> >> > Thanks,
>> >> >
>> >> > --
>> >> > Jianshi Huang
>> >> >
>> >> > LinkedIn: jianshi
>> >> > Twitter: @jshuang
>> >> > Github & Blog: http://huangjs.github.com/
>> >> >
>> >>
>> >
>> >
>> >
>> >--
>> >Jianshi Huang
>> >
>> >LinkedIn: jianshi
>> >Twitter: @jshuang
>> >Github & Blog: http://huangjs.github.com/
>>
>>
>
>
>-- 
>Jianshi Huang
>
>LinkedIn: jianshi
>Twitter: @jshuang
>Github & Blog: http://huangjs.github.com/


Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Thanks Ted, I'll try to do a major compact.

Hi Steven,

Yes, most of my rows are hashed to make it randomly distributed, but one
column family has monotonically increasing rowkeys, and it's used for
recording sequence of events.

Do you have a solution how to bulk import this kind of data?

Jianshi



On Sat, Sep 6, 2014 at 12:00 AM, Magana-zook, Steven Alan <
maganazook1@llnl.gov> wrote:

> Hi Jianshi,
>
> What are the field(s) in your row key? If your row key is monotonically
> increasing then you will be sending all of your requests to one region
> server. Even after the region splits, all new entries will keep punishing
> one server (the region responsible for the split containing the new keys).
>
> See these articles that may help if this is indeed your issue:
> 1. http://hbase.apache.org/book/rowkey.design.html
> 2.
> http://ikaisays.com/2011/01/25/app-engine-datastore-tip-monotonically-incre
> asing-values-are-bad/
>
> Regards,
> Steven Magana-Zook
>
>
>
>
>
>
> On 9/5/14 8:54 AM, "Jianshi Huang" <ji...@gmail.com> wrote:
>
> >Hi JM,
> >
> >What do you mean by the 'destination cluster'? The files are in the same
> >Hadoop/HDFS cluster where HBase is running.
> >
> >Do you mean do the bulk importing on HBase Master node?
> >
> >
> >Jianshi
> >
> >
> >On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
> >jean-marc@spaggiari.org> wrote:
> >
> >> Hi Jianshi,
> >>
> >> You might want to upload the file on the destination cluster first and
> >>then
> >> re-run your bulk load from there. That way the transfer time will not be
> >> taken into consideration for the timeout size the files will be local.
> >>
> >> JM
> >>
> >>
> >> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
> >>
> >> > I'm importing 2TB of generated HFiles to HBase and I constantly get
> >>the
> >> > following errors:
> >> >
> >> > Caused by:
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.h
> >>base.RegionTooBusyException):
> >> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock
> >>in
> >> > 60000 ms.
> >> >
> >> >
> >>
> >>regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f
> >>3565642d96.,
> >> > server=xxxxx.xxx.xxx,60020,1404854700728
> >> >         at
> >> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
> >> >         at
> >> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRe
> >>gion.java:5795)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
> >>3543)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
> >>3525)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionS
> >>erver.java:3277)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.c
> >>allBlockingMethod(ClientProtos.java:28863)
> >> >         at
> >> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
> >> >         at
> >>org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcSche
> >>duler.java:160)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcSchedu
> >>ler.java:38)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.j
> >>ava:110)
> >> >         at java.lang.Thread.run(Thread.java:724)
> >> >
> >> >         at
> >> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1
> >>684)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.ca
> >>llBlockingMethod(RpcClient.java:1737)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$Blo
> >>ckingStub.bulkLoadHFile(ClientProtos.java:29276)
> >> >         at
> >> >
> >> >
> >>
> >>org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.
> >>java:1548)
> >> >         ... 11 more
> >> >
> >> >
> >> > What makes the region too busy? Is there a way to improve it?
> >> >
> >> > Does that also mean some part of my data are not correctly imported?
> >> >
> >> >
> >> > Thanks,
> >> >
> >> > --
> >> > Jianshi Huang
> >> >
> >> > LinkedIn: jianshi
> >> > Twitter: @jshuang
> >> > Github & Blog: http://huangjs.github.com/
> >> >
> >>
> >
> >
> >
> >--
> >Jianshi Huang
> >
> >LinkedIn: jianshi
> >Twitter: @jshuang
> >Github & Blog: http://huangjs.github.com/
>
>


-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by "Magana-zook, Steven Alan" <ma...@llnl.gov>.
Hi Jianshi,

What are the field(s) in your row key? If your row key is monotonically
increasing then you will be sending all of your requests to one region
server. Even after the region splits, all new entries will keep punishing
one server (the region responsible for the split containing the new keys).

See these articles that may help if this is indeed your issue:
1. http://hbase.apache.org/book/rowkey.design.html
2. 
http://ikaisays.com/2011/01/25/app-engine-datastore-tip-monotonically-incre
asing-values-are-bad/

Regards,
Steven Magana-Zook






On 9/5/14 8:54 AM, "Jianshi Huang" <ji...@gmail.com> wrote:

>Hi JM,
>
>What do you mean by the 'destination cluster'? The files are in the same
>Hadoop/HDFS cluster where HBase is running.
>
>Do you mean do the bulk importing on HBase Master node?
>
>
>Jianshi
>
>
>On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
>jean-marc@spaggiari.org> wrote:
>
>> Hi Jianshi,
>>
>> You might want to upload the file on the destination cluster first and
>>then
>> re-run your bulk load from there. That way the transfer time will not be
>> taken into consideration for the timeout size the files will be local.
>>
>> JM
>>
>>
>> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
>>
>> > I'm importing 2TB of generated HFiles to HBase and I constantly get
>>the
>> > following errors:
>> >
>> > Caused by:
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.h
>>base.RegionTooBusyException):
>> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock
>>in
>> > 60000 ms.
>> >
>> >
>> 
>>regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f
>>3565642d96.,
>> > server=xxxxx.xxx.xxx,60020,1404854700728
>> >         at
>> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
>> >         at
>> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRe
>>gion.java:5795)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
>>3543)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:
>>3525)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionS
>>erver.java:3277)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.c
>>allBlockingMethod(ClientProtos.java:28863)
>> >         at
>> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
>> >         at 
>>org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcSche
>>duler.java:160)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcSchedu
>>ler.java:38)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.j
>>ava:110)
>> >         at java.lang.Thread.run(Thread.java:724)
>> >
>> >         at
>> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1
>>684)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.ca
>>llBlockingMethod(RpcClient.java:1737)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$Blo
>>ckingStub.bulkLoadHFile(ClientProtos.java:29276)
>> >         at
>> >
>> >
>> 
>>org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.
>>java:1548)
>> >         ... 11 more
>> >
>> >
>> > What makes the region too busy? Is there a way to improve it?
>> >
>> > Does that also mean some part of my data are not correctly imported?
>> >
>> >
>> > Thanks,
>> >
>> > --
>> > Jianshi Huang
>> >
>> > LinkedIn: jianshi
>> > Twitter: @jshuang
>> > Github & Blog: http://huangjs.github.com/
>> >
>>
>
>
>
>-- 
>Jianshi Huang
>
>LinkedIn: jianshi
>Twitter: @jshuang
>Github & Blog: http://huangjs.github.com/


Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jean-Marc Spaggiari <je...@spaggiari.org>.
If you files are on the same cluster, then this is not the issue. Do you
have the command line you run to launch your bulk load?



2014-09-05 11:54 GMT-04:00 Jianshi Huang <ji...@gmail.com>:

> Hi JM,
>
> What do you mean by the 'destination cluster'? The files are in the same
> Hadoop/HDFS cluster where HBase is running.
>
> Do you mean do the bulk importing on HBase Master node?
>
>
> Jianshi
>
>
> On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
> jean-marc@spaggiari.org> wrote:
>
> > Hi Jianshi,
> >
> > You might want to upload the file on the destination cluster first and
> then
> > re-run your bulk load from there. That way the transfer time will not be
> > taken into consideration for the timeout size the files will be local.
> >
> > JM
> >
> >
> > 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
> >
> > > I'm importing 2TB of generated HFiles to HBase and I constantly get the
> > > following errors:
> > >
> > > Caused by:
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.hbase.RegionTooBusyException):
> > > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock in
> > > 60000 ms.
> > >
> > >
> >
> regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f3565642d96.,
> > > server=xxxxx.xxx.xxx,60020,1404854700728
> > >         at
> > > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
> > >         at
> > > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRegion.java:5795)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3543)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3525)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionServer.java:3277)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.callBlockingMethod(ClientProtos.java:28863)
> > >         at
> > org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
> > >         at
> org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcScheduler.java:160)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcScheduler.java:38)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.java:110)
> > >         at java.lang.Thread.run(Thread.java:724)
> > >
> > >         at
> > org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1684)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.callBlockingMethod(RpcClient.java:1737)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$BlockingStub.bulkLoadHFile(ClientProtos.java:29276)
> > >         at
> > >
> > >
> >
> org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.java:1548)
> > >         ... 11 more
> > >
> > >
> > > What makes the region too busy? Is there a way to improve it?
> > >
> > > Does that also mean some part of my data are not correctly imported?
> > >
> > >
> > > Thanks,
> > >
> > > --
> > > Jianshi Huang
> > >
> > > LinkedIn: jianshi
> > > Twitter: @jshuang
> > > Github & Blog: http://huangjs.github.com/
> > >
> >
>
>
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jianshi Huang <ji...@gmail.com>.
Hi JM,

What do you mean by the 'destination cluster'? The files are in the same
Hadoop/HDFS cluster where HBase is running.

Do you mean do the bulk importing on HBase Master node?


Jianshi


On Fri, Sep 5, 2014 at 11:18 PM, Jean-Marc Spaggiari <
jean-marc@spaggiari.org> wrote:

> Hi Jianshi,
>
> You might want to upload the file on the destination cluster first and then
> re-run your bulk load from there. That way the transfer time will not be
> taken into consideration for the timeout size the files will be local.
>
> JM
>
>
> 2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:
>
> > I'm importing 2TB of generated HFiles to HBase and I constantly get the
> > following errors:
> >
> > Caused by:
> >
> >
> org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.hbase.RegionTooBusyException):
> > org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock in
> > 60000 ms.
> >
> >
> regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f3565642d96.,
> > server=xxxxx.xxx.xxx,60020,1404854700728
> >         at
> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
> >         at
> > org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
> >         at
> >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRegion.java:5795)
> >         at
> >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3543)
> >         at
> >
> >
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3525)
> >         at
> >
> >
> org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionServer.java:3277)
> >         at
> >
> >
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.callBlockingMethod(ClientProtos.java:28863)
> >         at
> org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
> >         at org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
> >         at
> >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcScheduler.java:160)
> >         at
> >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcScheduler.java:38)
> >         at
> >
> >
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.java:110)
> >         at java.lang.Thread.run(Thread.java:724)
> >
> >         at
> org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
> >         at
> >
> >
> org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1684)
> >         at
> >
> >
> org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.callBlockingMethod(RpcClient.java:1737)
> >         at
> >
> >
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$BlockingStub.bulkLoadHFile(ClientProtos.java:29276)
> >         at
> >
> >
> org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.java:1548)
> >         ... 11 more
> >
> >
> > What makes the region too busy? Is there a way to improve it?
> >
> > Does that also mean some part of my data are not correctly imported?
> >
> >
> > Thanks,
> >
> > --
> > Jianshi Huang
> >
> > LinkedIn: jianshi
> > Twitter: @jshuang
> > Github & Blog: http://huangjs.github.com/
> >
>



-- 
Jianshi Huang

LinkedIn: jianshi
Twitter: @jshuang
Github & Blog: http://huangjs.github.com/

Re: Help: RegionTooBusyException: failed to get a lock in 60000 ms

Posted by Jean-Marc Spaggiari <je...@spaggiari.org>.
Hi Jianshi,

You might want to upload the file on the destination cluster first and then
re-run your bulk load from there. That way the transfer time will not be
taken into consideration for the timeout size the files will be local.

JM


2014-09-05 11:15 GMT-04:00 Jianshi Huang <ji...@gmail.com>:

> I'm importing 2TB of generated HFiles to HBase and I constantly get the
> following errors:
>
> Caused by:
>
> org.apache.hadoop.hbase.ipc.RemoteWithExtrasException(org.apache.hadoop.hbase.RegionTooBusyException):
> org.apache.hadoop.hbase.RegionTooBusyException: failed to get a lock in
> 60000 ms.
>
> regionName=grapple_edges_v2,ff000000,1409817320781.6d2955c780b39523de733f3565642d96.,
> server=xxxxx.xxx.xxx,60020,1404854700728
>         at
> org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5851)
>         at
> org.apache.hadoop.hbase.regionserver.HRegion.lock(HRegion.java:5837)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.startBulkRegionOperation(HRegion.java:5795)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3543)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegion.bulkLoadHFiles(HRegion.java:3525)
>         at
>
> org.apache.hadoop.hbase.regionserver.HRegionServer.bulkLoadHFile(HRegionServer.java:3277)
>         at
>
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$2.callBlockingMethod(ClientProtos.java:28863)
>         at org.apache.hadoop.hbase.ipc.RpcServer.call(RpcServer.java:2008)
>         at org.apache.hadoop.hbase.ipc.CallRunner.run(CallRunner.java:92)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.consumerLoop(SimpleRpcScheduler.java:160)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler.access$000(SimpleRpcScheduler.java:38)
>         at
>
> org.apache.hadoop.hbase.ipc.SimpleRpcScheduler$1.run(SimpleRpcScheduler.java:110)
>         at java.lang.Thread.run(Thread.java:724)
>
>         at org.apache.hadoop.hbase.ipc.RpcClient.call(RpcClient.java:1498)
>         at
>
> org.apache.hadoop.hbase.ipc.RpcClient.callBlockingMethod(RpcClient.java:1684)
>         at
>
> org.apache.hadoop.hbase.ipc.RpcClient$BlockingRpcChannelImplementation.callBlockingMethod(RpcClient.java:1737)
>         at
>
> org.apache.hadoop.hbase.protobuf.generated.ClientProtos$ClientService$BlockingStub.bulkLoadHFile(ClientProtos.java:29276)
>         at
>
> org.apache.hadoop.hbase.protobuf.ProtobufUtil.bulkLoadHFile(ProtobufUtil.java:1548)
>         ... 11 more
>
>
> What makes the region too busy? Is there a way to improve it?
>
> Does that also mean some part of my data are not correctly imported?
>
>
> Thanks,
>
> --
> Jianshi Huang
>
> LinkedIn: jianshi
> Twitter: @jshuang
> Github & Blog: http://huangjs.github.com/
>