You are viewing a plain text version of this content. The canonical link for it is here.
Posted to mapreduce-user@hadoop.apache.org by "Liu, Raymond" <ra...@intel.com> on 2013/02/16 04:10:14 UTC

why my test result on dfs short circuit read is slower?

Hi

	I tried to use short circuit read to improve my hbase cluster MR scan performance.

	I have the following setting in hdfs-site.xml

	dfs.client.read.shortcircuit set to true
	dfs.block.local-path-access.user set to MR job runner.

	The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
	I have hoped that the short circuit read will improve the performance.

	While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.

	My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu



答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
I'm not very clear about your senario, just a kindly reminder: "If security is on, the feature can be used only for user that has kerberos credentials at the client, therefore map reduce tasks cannot benefit from it in general",  see HDFS-2246's release note for more info
If you didn't enable security at all, please ignore my comments:)

Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:40
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


>
> Hi Raymond,
>
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
>
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
>
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
>
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
I'm not very clear about your senario, just a kindly reminder: "If security is on, the feature can be used only for user that has kerberos credentials at the client, therefore map reduce tasks cannot benefit from it in general",  see HDFS-2246's release note for more info
If you didn't enable security at all, please ignore my comments:)

Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:40
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


>
> Hi Raymond,
>
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
>
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
>
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
>
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
I'm not very clear about your senario, just a kindly reminder: "If security is on, the feature can be used only for user that has kerberos credentials at the client, therefore map reduce tasks cannot benefit from it in general",  see HDFS-2246's release note for more info
If you didn't enable security at all, please ignore my comments:)

Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:40
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


>
> Hi Raymond,
>
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
>
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
>
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
>
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
I'm not very clear about your senario, just a kindly reminder: "If security is on, the feature can be used only for user that has kerberos credentials at the client, therefore map reduce tasks cannot benefit from it in general",  see HDFS-2246's release note for more info
If you didn't enable security at all, please ignore my comments:)

Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:40
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


>
> Hi Raymond,
>
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
>
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
>
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
>
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


> 
> Hi Raymond,
> 
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
> 
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
> 
> Hi
> 
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
>         I have the following setting in hdfs-site.xml
> 
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
> 
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
> 
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
>         My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


> 
> Hi Raymond,
> 
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
> 
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
> 
> Hi
> 
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
>         I have the following setting in hdfs-site.xml
> 
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
> 
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
> 
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
>         My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


> 
> Hi Raymond,
> 
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
> 
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
> 
> Hi
> 
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
>         I have the following setting in hdfs-site.xml
> 
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
> 
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
> 
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
>         My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Liang

Did you mean set dfs.permissions to false?

Is that all I need to do to disable security feature? Cause It seems to me that without change dfs.block.local-path-access.user, dfs.permissions alone doesn't works. HBASE still fall back to go through datanode to read data.


> 
> Hi Raymond,
> 
> did you enable security feature in your cluster?  there'll be no obvious benefit
> be found if so.
> 
> Regards,
> Liang
> _______________________________________
> 发件人: Liu, Raymond [raymond.liu@intel.com]
> 发送时间: 2013年2月16日 11:10
> 收件人: user@hadoop.apache.org
> 主题: why my test result on dfs short circuit read is slower?
> 
> Hi
> 
>         I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
>         I have the following setting in hdfs-site.xml
> 
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
> 
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the
> performance.
> 
>         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
>         My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Hi Raymond,

did you enable security feature in your cluster?  there'll be no obvious benefit be found if so.

Regards,
Liang
_______________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:10
收件人: user@hadoop.apache.org
主题: why my test result on dfs short circuit read is slower?

Hi

        I tried to use short circuit read to improve my hbase cluster MR scan performance.

        I have the following setting in hdfs-site.xml

        dfs.client.read.shortcircuit set to true
        dfs.block.local-path-access.user set to MR job runner.

        The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
        I have hoped that the short circuit read will improve the performance.

        While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.

        My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Hi Raymond,

did you enable security feature in your cluster?  there'll be no obvious benefit be found if so.

Regards,
Liang
_______________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:10
收件人: user@hadoop.apache.org
主题: why my test result on dfs short circuit read is slower?

Hi

        I tried to use short circuit read to improve my hbase cluster MR scan performance.

        I have the following setting in hdfs-site.xml

        dfs.client.read.shortcircuit set to true
        dfs.block.local-path-access.user set to MR job runner.

        The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
        I have hoped that the short circuit read will improve the performance.

        While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.

        My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu

Re: why my test result on dfs short circuit read is slower?

Posted by Michael Segel <mi...@hotmail.com>.
On Feb 17, 2013, at 7:09 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> io.file.buffer.size


Drop this down to 64KB not 128KB. 

You have 16 cpu which really means 8 cores and 4 disks. 

Do you have Ganglia up and running? 

I'll wager that you'll see a lot of wait cpu cycles in both cases. 

Also how much memory do you have on each machine? 


Tuning is going to be hardware specific and without really understanding what each parameter does, you can hurt performance. 

Michael Segel  | (m) 312.755.9623

Segel and Associates




The opinions expressed here are mine, while they may reflect a cognitive thought, that is purely accidental. 
Use at your own risk. 
Michael Segel
michael_segel (AT) hotmail.com






答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Probably readahead played a key role on the first scenario(scan only job) ?  the default LONG_READ_THRESHOLD_BYTES(BlockSender.java) is 256k in current codebase, and ReadaheadPool takes effect on normal read path.


Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月18日 14:04
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load.

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

>
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
>
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
>
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Probably readahead played a key role on the first scenario(scan only job) ?  the default LONG_READ_THRESHOLD_BYTES(BlockSender.java) is 256k in current codebase, and ReadaheadPool takes effect on normal read path.


Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月18日 14:04
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load.

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

>
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
>
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
>
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Probably readahead played a key role on the first scenario(scan only job) ?  the default LONG_READ_THRESHOLD_BYTES(BlockSender.java) is 256k in current codebase, and ReadaheadPool takes effect on normal read path.


Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月18日 14:04
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load.

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

>
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
>
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
>
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Probably readahead played a key role on the first scenario(scan only job) ?  the default LONG_READ_THRESHOLD_BYTES(BlockSender.java) is 256k in current codebase, and ReadaheadPool takes effect on normal read path.


Regards,
Liang
________________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月18日 14:04
收件人: user@hadoop.apache.org
主题: RE: why my test result on dfs short circuit read is slower?

Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load.

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

>
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
>
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
>
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load. 

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

> 
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
> 
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
> 
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load. 

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

> 
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
> 
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
> 
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Michael Segel <mi...@hotmail.com>.
On Feb 17, 2013, at 7:09 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> io.file.buffer.size


Drop this down to 64KB not 128KB. 

You have 16 cpu which really means 8 cores and 4 disks. 

Do you have Ganglia up and running? 

I'll wager that you'll see a lot of wait cpu cycles in both cases. 

Also how much memory do you have on each machine? 


Tuning is going to be hardware specific and without really understanding what each parameter does, you can hurt performance. 

Michael Segel  | (m) 312.755.9623

Segel and Associates




The opinions expressed here are mine, while they may reflect a cognitive thought, that is purely accidental. 
Use at your own risk. 
Michael Segel
michael_segel (AT) hotmail.com






RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load. 

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

> 
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
> 
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
> 
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Alright, I think in my sequence read scenario, it is possible that shortcircuit read is actually slower than read through datanode.

For, when read through datanode, FS read operation is done by datanode daemon, while data processing is done by client.
Thus when client is processing the data, data node could read data at the same time and write it to local socket for client to read.
It take very few time for client to read from local socket.

While when client read native FS directly, all the job is done by client. It will be blocked for more time to read data from native FS for process than from local socket.

Overall, when CPU is not bound, and data node always prepare further data for client(due to sequence read scenario), the result is that shortcircuit read is slower though it cost less CPU load. 

The CPU/idle/IOWait load seems also justify my guess

= For a scan only job: =

Read through datanode:  CPU : 30-35% / IOWait : ~50% / 300 second
Shortcircuit read:  CPU : 25-30% / IOWait : ~40% / 330 second
Short circuit read is 10% slower

= For a job do more calculation =
Read through datanode:  CPU : 80-90% / IOWait : ~5-10% / 190 seconds
Shortcircuit read:  CPU : ~90% / IOWait : ~2-3% / 160 second
Short circuit read is 15% faster.

So, short circuit read is not always faster, especially when CPU is not bound and read by sequence, it will be slower. This is the best explain I can get now.
Any thoughts?

Raymond

> 
> I have try to tune io.file.buffer.size to 128K instead of 4K ShortCircuit read
> performance is still worse than read through datanode.
> 
> I am start to wondering, does shortcircuit read really help under hadoop 1.1.1
> version?
> I google to find a few people mention they got 2x gain or so upon CDH etc. But I
> really can't find out what else I can do to make it even just catch up normal
> read path....
> 
> >
> > It seems to me that, with short circuit read enabled, the
> > BlockReaderLocal read data in 512/4096 bytes unit(checksum check
> > enabled/skiped)
> >
> > While when It go through datanode, the BlockSender.sendChunks will
> > read and sent data in 64K bytes units?
> >
> > Is that true? And if so, won't it explain that read through datanode
> > will be faster? Since it read data in bigger block size.
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> > > -----Original Message-----
> > > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > > Sent: Saturday, February 16, 2013 2:23 PM
> > > To: user@hadoop.apache.org
> > > Subject: RE: why my test result on dfs short circuit read is slower?
> > >
> > > Hi Arpit Gupta
> > >
> > > Yes,  this way also confirms that short circuit read is enabled on my
> cluster.
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> > >
> > > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for
> > > file
> > >
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > > checksum false
> > >
> > > So , any possibility that other setting might impact short circuit
> > > read to has worse performance than read through datanode?
> > >
> > > Raymond
> > >
> > >
> > >
> > > >Another way to check if short circuit read is configured correctly.
> > >
> > > >As the user who is configured for short circuit read issue the
> > > >following
> > > command on a node where you expect the data to be read locally.
> > >
> > > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > > >/path/to/file_on_hdfs
> > >
> > > >On the console you should see something like "hdfs.DFSClient: New
> > > BlockReaderLocal for file...."
> > >
> > > >This would confirm that short circuit read is happening.
> > >
> > > --
> > > >Arpit Gupta
> > > >Hortonworks Inc.
> > > >http://hortonworks.com/
> > >
> > > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> > wrote:
> > >
> > >
> > > Hi Harsh
> > >
> > > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> > >
> > > And I have double confirmed that local reads are performed, since
> > > there are no Error in datanode logs, and by watching lo network IO.
> > >
> > >
> > >
> > > If you want HBase to leverage the shortcircuit, the DN config
> > > "dfs.block.local-path-access.user" should be set to the user running
> > > HBase
> > (i.e.
> > > hbase, for example), and the hbase-site.xml should have
> > > "dfs.client.read.shortcircuit" defined in all its RegionServers.
> > > Doing this wrong could result in performance penalty and some
> > > warn-logging, as local reads will be attempted but will begin to fail.
> > >
> > > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond
> > > <ra...@intel.com>
> > > wrote:
> > >
> > > Hi
> > >
> > >        I tried to use short circuit read to improve my hbase cluster
> > > MR scan performance.
> > >
> > >
> > >        I have the following setting in hdfs-site.xml
> > >
> > >        dfs.client.read.shortcircuit set to true
> > >        dfs.block.local-path-access.user set to MR job runner.
> > >
> > >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > > with all hbase table major compact thus all data is local.
> > >
> > >        I have hoped that the short circuit read will improve the
> > > performance.
> > >
> > >
> > >        While the test result is that with short circuit read
> > > enabled, the performance actually dropped 10-15%. Say scan a 50G
> > > table cost around 100s instead of 90s.
> > >
> > >
> > >        My hadoop version is 1.1.1, any idea on this? Thx!
> > >
> > > Best Regards,
> > > Raymond Liu
> > >
> > >
> > >
> > >
> > > --
> > > Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Michael Segel <mi...@hotmail.com>.
On Feb 17, 2013, at 7:09 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> io.file.buffer.size


Drop this down to 64KB not 128KB. 

You have 16 cpu which really means 8 cores and 4 disks. 

Do you have Ganglia up and running? 

I'll wager that you'll see a lot of wait cpu cycles in both cases. 

Also how much memory do you have on each machine? 


Tuning is going to be hardware specific and without really understanding what each parameter does, you can hurt performance. 

Michael Segel  | (m) 312.755.9623

Segel and Associates




The opinions expressed here are mine, while they may reflect a cognitive thought, that is purely accidental. 
Use at your own risk. 
Michael Segel
michael_segel (AT) hotmail.com






Re: why my test result on dfs short circuit read is slower?

Posted by Michael Segel <mi...@hotmail.com>.
On Feb 17, 2013, at 7:09 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> io.file.buffer.size


Drop this down to 64KB not 128KB. 

You have 16 cpu which really means 8 cores and 4 disks. 

Do you have Ganglia up and running? 

I'll wager that you'll see a lot of wait cpu cycles in both cases. 

Also how much memory do you have on each machine? 


Tuning is going to be hardware specific and without really understanding what each parameter does, you can hurt performance. 

Michael Segel  | (m) 312.755.9623

Segel and Associates




The opinions expressed here are mine, while they may reflect a cognitive thought, that is purely accidental. 
Use at your own risk. 
Michael Segel
michael_segel (AT) hotmail.com






RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
I have try to tune io.file.buffer.size to 128K instead of 4K
ShortCircuit read performance is still worse than read through datanode.

I am start to wondering, does shortcircuit read really help under hadoop 1.1.1 version?
I google to find a few people mention they got 2x gain or so upon CDH etc. But I really can't find out what else I can do to make it even just catch up normal read path....

> 
> It seems to me that, with short circuit read enabled, the BlockReaderLocal
> read data in 512/4096 bytes unit(checksum check enabled/skiped)
> 
> While when It go through datanode, the BlockSender.sendChunks will read and
> sent data in 64K bytes units?
> 
> Is that true? And if so, won't it explain that read through datanode will be
> faster? Since it read data in bigger block size.
> 
> Best Regards,
> Raymond Liu
> 
> 
> > -----Original Message-----
> > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > Sent: Saturday, February 16, 2013 2:23 PM
> > To: user@hadoop.apache.org
> > Subject: RE: why my test result on dfs short circuit read is slower?
> >
> > Hi Arpit Gupta
> >
> > Yes,  this way also confirms that short circuit read is enabled on my cluster.
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > checksum false
> >
> > So , any possibility that other setting might impact short circuit
> > read to has worse performance than read through datanode?
> >
> > Raymond
> >
> >
> >
> > >Another way to check if short circuit read is configured correctly.
> >
> > >As the user who is configured for short circuit read issue the
> > >following
> > command on a node where you expect the data to be read locally.
> >
> > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > >/path/to/file_on_hdfs
> >
> > >On the console you should see something like "hdfs.DFSClient: New
> > BlockReaderLocal for file...."
> >
> > >This would confirm that short circuit read is happening.
> >
> > --
> > >Arpit Gupta
> > >Hortonworks Inc.
> > >http://hortonworks.com/
> >
> > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> wrote:
> >
> >
> > Hi Harsh
> >
> > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> >
> > And I have double confirmed that local reads are performed, since
> > there are no Error in datanode logs, and by watching lo network IO.
> >
> >
> >
> > If you want HBase to leverage the shortcircuit, the DN config
> > "dfs.block.local-path-access.user" should be set to the user running HBase
> (i.e.
> > hbase, for example), and the hbase-site.xml should have
> > "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
> > this wrong could result in performance penalty and some warn-logging,
> > as local reads will be attempted but will begin to fail.
> >
> > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> > wrote:
> >
> > Hi
> >
> >        I tried to use short circuit read to improve my hbase cluster
> > MR scan performance.
> >
> >
> >        I have the following setting in hdfs-site.xml
> >
> >        dfs.client.read.shortcircuit set to true
> >        dfs.block.local-path-access.user set to MR job runner.
> >
> >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > with all hbase table major compact thus all data is local.
> >
> >        I have hoped that the short circuit read will improve the
> > performance.
> >
> >
> >        While the test result is that with short circuit read enabled,
> > the performance actually dropped 10-15%. Say scan a 50G table cost
> > around 100s instead of 90s.
> >
> >
> >        My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> >
> >
> > --
> > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
I have try to tune io.file.buffer.size to 128K instead of 4K
ShortCircuit read performance is still worse than read through datanode.

I am start to wondering, does shortcircuit read really help under hadoop 1.1.1 version?
I google to find a few people mention they got 2x gain or so upon CDH etc. But I really can't find out what else I can do to make it even just catch up normal read path....

> 
> It seems to me that, with short circuit read enabled, the BlockReaderLocal
> read data in 512/4096 bytes unit(checksum check enabled/skiped)
> 
> While when It go through datanode, the BlockSender.sendChunks will read and
> sent data in 64K bytes units?
> 
> Is that true? And if so, won't it explain that read through datanode will be
> faster? Since it read data in bigger block size.
> 
> Best Regards,
> Raymond Liu
> 
> 
> > -----Original Message-----
> > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > Sent: Saturday, February 16, 2013 2:23 PM
> > To: user@hadoop.apache.org
> > Subject: RE: why my test result on dfs short circuit read is slower?
> >
> > Hi Arpit Gupta
> >
> > Yes,  this way also confirms that short circuit read is enabled on my cluster.
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > checksum false
> >
> > So , any possibility that other setting might impact short circuit
> > read to has worse performance than read through datanode?
> >
> > Raymond
> >
> >
> >
> > >Another way to check if short circuit read is configured correctly.
> >
> > >As the user who is configured for short circuit read issue the
> > >following
> > command on a node where you expect the data to be read locally.
> >
> > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > >/path/to/file_on_hdfs
> >
> > >On the console you should see something like "hdfs.DFSClient: New
> > BlockReaderLocal for file...."
> >
> > >This would confirm that short circuit read is happening.
> >
> > --
> > >Arpit Gupta
> > >Hortonworks Inc.
> > >http://hortonworks.com/
> >
> > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> wrote:
> >
> >
> > Hi Harsh
> >
> > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> >
> > And I have double confirmed that local reads are performed, since
> > there are no Error in datanode logs, and by watching lo network IO.
> >
> >
> >
> > If you want HBase to leverage the shortcircuit, the DN config
> > "dfs.block.local-path-access.user" should be set to the user running HBase
> (i.e.
> > hbase, for example), and the hbase-site.xml should have
> > "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
> > this wrong could result in performance penalty and some warn-logging,
> > as local reads will be attempted but will begin to fail.
> >
> > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> > wrote:
> >
> > Hi
> >
> >        I tried to use short circuit read to improve my hbase cluster
> > MR scan performance.
> >
> >
> >        I have the following setting in hdfs-site.xml
> >
> >        dfs.client.read.shortcircuit set to true
> >        dfs.block.local-path-access.user set to MR job runner.
> >
> >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > with all hbase table major compact thus all data is local.
> >
> >        I have hoped that the short circuit read will improve the
> > performance.
> >
> >
> >        While the test result is that with short circuit read enabled,
> > the performance actually dropped 10-15%. Say scan a 50G table cost
> > around 100s instead of 90s.
> >
> >
> >        My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> >
> >
> > --
> > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
I have try to tune io.file.buffer.size to 128K instead of 4K
ShortCircuit read performance is still worse than read through datanode.

I am start to wondering, does shortcircuit read really help under hadoop 1.1.1 version?
I google to find a few people mention they got 2x gain or so upon CDH etc. But I really can't find out what else I can do to make it even just catch up normal read path....

> 
> It seems to me that, with short circuit read enabled, the BlockReaderLocal
> read data in 512/4096 bytes unit(checksum check enabled/skiped)
> 
> While when It go through datanode, the BlockSender.sendChunks will read and
> sent data in 64K bytes units?
> 
> Is that true? And if so, won't it explain that read through datanode will be
> faster? Since it read data in bigger block size.
> 
> Best Regards,
> Raymond Liu
> 
> 
> > -----Original Message-----
> > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > Sent: Saturday, February 16, 2013 2:23 PM
> > To: user@hadoop.apache.org
> > Subject: RE: why my test result on dfs short circuit read is slower?
> >
> > Hi Arpit Gupta
> >
> > Yes,  this way also confirms that short circuit read is enabled on my cluster.
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > checksum false
> >
> > So , any possibility that other setting might impact short circuit
> > read to has worse performance than read through datanode?
> >
> > Raymond
> >
> >
> >
> > >Another way to check if short circuit read is configured correctly.
> >
> > >As the user who is configured for short circuit read issue the
> > >following
> > command on a node where you expect the data to be read locally.
> >
> > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > >/path/to/file_on_hdfs
> >
> > >On the console you should see something like "hdfs.DFSClient: New
> > BlockReaderLocal for file...."
> >
> > >This would confirm that short circuit read is happening.
> >
> > --
> > >Arpit Gupta
> > >Hortonworks Inc.
> > >http://hortonworks.com/
> >
> > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> wrote:
> >
> >
> > Hi Harsh
> >
> > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> >
> > And I have double confirmed that local reads are performed, since
> > there are no Error in datanode logs, and by watching lo network IO.
> >
> >
> >
> > If you want HBase to leverage the shortcircuit, the DN config
> > "dfs.block.local-path-access.user" should be set to the user running HBase
> (i.e.
> > hbase, for example), and the hbase-site.xml should have
> > "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
> > this wrong could result in performance penalty and some warn-logging,
> > as local reads will be attempted but will begin to fail.
> >
> > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> > wrote:
> >
> > Hi
> >
> >        I tried to use short circuit read to improve my hbase cluster
> > MR scan performance.
> >
> >
> >        I have the following setting in hdfs-site.xml
> >
> >        dfs.client.read.shortcircuit set to true
> >        dfs.block.local-path-access.user set to MR job runner.
> >
> >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > with all hbase table major compact thus all data is local.
> >
> >        I have hoped that the short circuit read will improve the
> > performance.
> >
> >
> >        While the test result is that with short circuit read enabled,
> > the performance actually dropped 10-15%. Say scan a 50G table cost
> > around 100s instead of 90s.
> >
> >
> >        My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> >
> >
> > --
> > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
I have try to tune io.file.buffer.size to 128K instead of 4K
ShortCircuit read performance is still worse than read through datanode.

I am start to wondering, does shortcircuit read really help under hadoop 1.1.1 version?
I google to find a few people mention they got 2x gain or so upon CDH etc. But I really can't find out what else I can do to make it even just catch up normal read path....

> 
> It seems to me that, with short circuit read enabled, the BlockReaderLocal
> read data in 512/4096 bytes unit(checksum check enabled/skiped)
> 
> While when It go through datanode, the BlockSender.sendChunks will read and
> sent data in 64K bytes units?
> 
> Is that true? And if so, won't it explain that read through datanode will be
> faster? Since it read data in bigger block size.
> 
> Best Regards,
> Raymond Liu
> 
> 
> > -----Original Message-----
> > From: Liu, Raymond [mailto:raymond.liu@intel.com]
> > Sent: Saturday, February 16, 2013 2:23 PM
> > To: user@hadoop.apache.org
> > Subject: RE: why my test result on dfs short circuit read is slower?
> >
> > Hi Arpit Gupta
> >
> > Yes,  this way also confirms that short circuit read is enabled on my cluster.
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> >
> > 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> >
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> > 638 of size 134217728 startOffset 0 length 134217728 short circuit
> > checksum false
> >
> > So , any possibility that other setting might impact short circuit
> > read to has worse performance than read through datanode?
> >
> > Raymond
> >
> >
> >
> > >Another way to check if short circuit read is configured correctly.
> >
> > >As the user who is configured for short circuit read issue the
> > >following
> > command on a node where you expect the data to be read locally.
> >
> > >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> > >/path/to/file_on_hdfs
> >
> > >On the console you should see something like "hdfs.DFSClient: New
> > BlockReaderLocal for file...."
> >
> > >This would confirm that short circuit read is happening.
> >
> > --
> > >Arpit Gupta
> > >Hortonworks Inc.
> > >http://hortonworks.com/
> >
> > On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com>
> wrote:
> >
> >
> > Hi Harsh
> >
> > Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> >
> > And I have double confirmed that local reads are performed, since
> > there are no Error in datanode logs, and by watching lo network IO.
> >
> >
> >
> > If you want HBase to leverage the shortcircuit, the DN config
> > "dfs.block.local-path-access.user" should be set to the user running HBase
> (i.e.
> > hbase, for example), and the hbase-site.xml should have
> > "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
> > this wrong could result in performance penalty and some warn-logging,
> > as local reads will be attempted but will begin to fail.
> >
> > On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> > wrote:
> >
> > Hi
> >
> >        I tried to use short circuit read to improve my hbase cluster
> > MR scan performance.
> >
> >
> >        I have the following setting in hdfs-site.xml
> >
> >        dfs.client.read.shortcircuit set to true
> >        dfs.block.local-path-access.user set to MR job runner.
> >
> >        The cluster is 1+4 node and each data node have 16cpu/4HDD,
> > with all hbase table major compact thus all data is local.
> >
> >        I have hoped that the short circuit read will improve the
> > performance.
> >
> >
> >        While the test result is that with short circuit read enabled,
> > the performance actually dropped 10-15%. Say scan a 50G table cost
> > around 100s instead of 90s.
> >
> >
> >        My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> >
> >
> > --
> > Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
It seems to me that, with short circuit read enabled, the BlockReaderLocal read data in 512/4096 bytes unit(checksum check enabled/skiped)

While when It go through datanode, the BlockSender.sendChunks will read and sent data in 64K bytes units?

Is that true? And if so, won't it explain that read through datanode will be faster? Since it read data in bigger block size.

Best Regards,
Raymond Liu


> -----Original Message-----
> From: Liu, Raymond [mailto:raymond.liu@intel.com]
> Sent: Saturday, February 16, 2013 2:23 PM
> To: user@hadoop.apache.org
> Subject: RE: why my test result on dfs short circuit read is slower?
> 
> Hi Arpit Gupta
> 
> Yes,  this way also confirms that short circuit read is enabled on my cluster.
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> 638 of size 134217728 startOffset 0 length 134217728 short circuit checksum
> false
> 
> So , any possibility that other setting might impact short circuit read to has
> worse performance than read through datanode?
> 
> Raymond
> 
> 
> 
> >Another way to check if short circuit read is configured correctly.
> 
> >As the user who is configured for short circuit read issue the following
> command on a node where you expect the data to be read locally.
> 
> >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> >/path/to/file_on_hdfs
> 
> >On the console you should see something like "hdfs.DFSClient: New
> BlockReaderLocal for file...."
> 
> >This would confirm that short circuit read is happening.
> 
> --
> >Arpit Gupta
> >Hortonworks Inc.
> >http://hortonworks.com/
> 
> On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:
> 
> 
> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no
> Error in datanode logs, and by watching lo network IO.
> 
> 
> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> 
> Hi
> 
>        I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
> 
>        I have the following setting in hdfs-site.xml
> 
>        dfs.client.read.shortcircuit set to true
>        dfs.block.local-path-access.user set to MR job runner.
> 
>        The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
> 
>        I have hoped that the short circuit read will improve the
> performance.
> 
> 
>        While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
> 
>        My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu
> 
> 
> 
> 
> --
> Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
It seems to me that, with short circuit read enabled, the BlockReaderLocal read data in 512/4096 bytes unit(checksum check enabled/skiped)

While when It go through datanode, the BlockSender.sendChunks will read and sent data in 64K bytes units?

Is that true? And if so, won't it explain that read through datanode will be faster? Since it read data in bigger block size.

Best Regards,
Raymond Liu


> -----Original Message-----
> From: Liu, Raymond [mailto:raymond.liu@intel.com]
> Sent: Saturday, February 16, 2013 2:23 PM
> To: user@hadoop.apache.org
> Subject: RE: why my test result on dfs short circuit read is slower?
> 
> Hi Arpit Gupta
> 
> Yes,  this way also confirms that short circuit read is enabled on my cluster.
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> 638 of size 134217728 startOffset 0 length 134217728 short circuit checksum
> false
> 
> So , any possibility that other setting might impact short circuit read to has
> worse performance than read through datanode?
> 
> Raymond
> 
> 
> 
> >Another way to check if short circuit read is configured correctly.
> 
> >As the user who is configured for short circuit read issue the following
> command on a node where you expect the data to be read locally.
> 
> >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> >/path/to/file_on_hdfs
> 
> >On the console you should see something like "hdfs.DFSClient: New
> BlockReaderLocal for file...."
> 
> >This would confirm that short circuit read is happening.
> 
> --
> >Arpit Gupta
> >Hortonworks Inc.
> >http://hortonworks.com/
> 
> On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:
> 
> 
> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no
> Error in datanode logs, and by watching lo network IO.
> 
> 
> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> 
> Hi
> 
>        I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
> 
>        I have the following setting in hdfs-site.xml
> 
>        dfs.client.read.shortcircuit set to true
>        dfs.block.local-path-access.user set to MR job runner.
> 
>        The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
> 
>        I have hoped that the short circuit read will improve the
> performance.
> 
> 
>        While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
> 
>        My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu
> 
> 
> 
> 
> --
> Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
It seems to me that, with short circuit read enabled, the BlockReaderLocal read data in 512/4096 bytes unit(checksum check enabled/skiped)

While when It go through datanode, the BlockSender.sendChunks will read and sent data in 64K bytes units?

Is that true? And if so, won't it explain that read through datanode will be faster? Since it read data in bigger block size.

Best Regards,
Raymond Liu


> -----Original Message-----
> From: Liu, Raymond [mailto:raymond.liu@intel.com]
> Sent: Saturday, February 16, 2013 2:23 PM
> To: user@hadoop.apache.org
> Subject: RE: why my test result on dfs short circuit read is slower?
> 
> Hi Arpit Gupta
> 
> Yes,  this way also confirms that short circuit read is enabled on my cluster.
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> 638 of size 134217728 startOffset 0 length 134217728 short circuit checksum
> false
> 
> So , any possibility that other setting might impact short circuit read to has
> worse performance than read through datanode?
> 
> Raymond
> 
> 
> 
> >Another way to check if short circuit read is configured correctly.
> 
> >As the user who is configured for short circuit read issue the following
> command on a node where you expect the data to be read locally.
> 
> >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> >/path/to/file_on_hdfs
> 
> >On the console you should see something like "hdfs.DFSClient: New
> BlockReaderLocal for file...."
> 
> >This would confirm that short circuit read is happening.
> 
> --
> >Arpit Gupta
> >Hortonworks Inc.
> >http://hortonworks.com/
> 
> On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:
> 
> 
> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no
> Error in datanode logs, and by watching lo network IO.
> 
> 
> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> 
> Hi
> 
>        I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
> 
>        I have the following setting in hdfs-site.xml
> 
>        dfs.client.read.shortcircuit set to true
>        dfs.block.local-path-access.user set to MR job runner.
> 
>        The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
> 
>        I have hoped that the short circuit read will improve the
> performance.
> 
> 
>        While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
> 
>        My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu
> 
> 
> 
> 
> --
> Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
It seems to me that, with short circuit read enabled, the BlockReaderLocal read data in 512/4096 bytes unit(checksum check enabled/skiped)

While when It go through datanode, the BlockSender.sendChunks will read and sent data in 64K bytes units?

Is that true? And if so, won't it explain that read through datanode will be faster? Since it read data in bigger block size.

Best Regards,
Raymond Liu


> -----Original Message-----
> From: Liu, Raymond [mailto:raymond.liu@intel.com]
> Sent: Saturday, February 16, 2013 2:23 PM
> To: user@hadoop.apache.org
> Subject: RE: why my test result on dfs short circuit read is slower?
> 
> Hi Arpit Gupta
> 
> Yes,  this way also confirms that short circuit read is enabled on my cluster.
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true
> 
> 13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file
> /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727
> 638 of size 134217728 startOffset 0 length 134217728 short circuit checksum
> false
> 
> So , any possibility that other setting might impact short circuit read to has
> worse performance than read through datanode?
> 
> Raymond
> 
> 
> 
> >Another way to check if short circuit read is configured correctly.
> 
> >As the user who is configured for short circuit read issue the following
> command on a node where you expect the data to be read locally.
> 
> >export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat
> >/path/to/file_on_hdfs
> 
> >On the console you should see something like "hdfs.DFSClient: New
> BlockReaderLocal for file...."
> 
> >This would confirm that short circuit read is happening.
> 
> --
> >Arpit Gupta
> >Hortonworks Inc.
> >http://hortonworks.com/
> 
> On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:
> 
> 
> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no
> Error in datanode logs, and by watching lo network IO.
> 
> 
> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> 
> Hi
> 
>        I tried to use short circuit read to improve my hbase cluster MR scan
> performance.
> 
> 
>        I have the following setting in hdfs-site.xml
> 
>        dfs.client.read.shortcircuit set to true
>        dfs.block.local-path-access.user set to MR job runner.
> 
>        The cluster is 1+4 node and each data node have 16cpu/4HDD, with
> all hbase table major compact thus all data is local.
> 
>        I have hoped that the short circuit read will improve the
> performance.
> 
> 
>        While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> 
> 
>        My hadoop version is 1.1.1, any idea on this? Thx!
> 
> Best Regards,
> Raymond Liu
> 
> 
> 
> 
> --
> Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Arpit Gupta

Yes,  this way also confirms that short circuit read is enabled on my cluster.

13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true

13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727638 of size 134217728 startOffset 0 length 134217728 short circuit checksum false

So , any possibility that other setting might impact short circuit read to has worse performance than read through datanode?

Raymond



>Another way to check if short circuit read is configured correctly.

>As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

>export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

>On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

>This would confirm that short circuit read is happening.

--
>Arpit Gupta
>Hortonworks Inc.
>http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:


Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.



If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
could result in performance penalty and some warn-logging, as local reads will
be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
wrote:

Hi

       I tried to use short circuit read to improve my hbase cluster MR
scan performance.


       I have the following setting in hdfs-site.xml

       dfs.client.read.shortcircuit set to true
       dfs.block.local-path-access.user set to MR job runner.

       The cluster is 1+4 node and each data node have 16cpu/4HDD,
with all hbase table major compact thus all data is local.

       I have hoped that the short circuit read will improve the
performance.


       While the test result is that with short circuit read enabled, the
performance actually dropped 10-15%. Say scan a 50G table cost around 100s
instead of 90s.


       My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu




--
Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Arpit Gupta

Yes,  this way also confirms that short circuit read is enabled on my cluster.

13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true

13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727638 of size 134217728 startOffset 0 length 134217728 short circuit checksum false

So , any possibility that other setting might impact short circuit read to has worse performance than read through datanode?

Raymond



>Another way to check if short circuit read is configured correctly.

>As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

>export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

>On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

>This would confirm that short circuit read is happening.

--
>Arpit Gupta
>Hortonworks Inc.
>http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:


Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.



If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
could result in performance penalty and some warn-logging, as local reads will
be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
wrote:

Hi

       I tried to use short circuit read to improve my hbase cluster MR
scan performance.


       I have the following setting in hdfs-site.xml

       dfs.client.read.shortcircuit set to true
       dfs.block.local-path-access.user set to MR job runner.

       The cluster is 1+4 node and each data node have 16cpu/4HDD,
with all hbase table major compact thus all data is local.

       I have hoped that the short circuit read will improve the
performance.


       While the test result is that with short circuit read enabled, the
performance actually dropped 10-15%. Say scan a 50G table cost around 100s
instead of 90s.


       My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu




--
Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Arpit Gupta

Yes,  this way also confirms that short circuit read is enabled on my cluster.

13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true

13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727638 of size 134217728 startOffset 0 length 134217728 short circuit checksum false

So , any possibility that other setting might impact short circuit read to has worse performance than read through datanode?

Raymond



>Another way to check if short circuit read is configured correctly.

>As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

>export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

>On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

>This would confirm that short circuit read is happening.

--
>Arpit Gupta
>Hortonworks Inc.
>http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:


Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.



If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
could result in performance penalty and some warn-logging, as local reads will
be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
wrote:

Hi

       I tried to use short circuit read to improve my hbase cluster MR
scan performance.


       I have the following setting in hdfs-site.xml

       dfs.client.read.shortcircuit set to true
       dfs.block.local-path-access.user set to MR job runner.

       The cluster is 1+4 node and each data node have 16cpu/4HDD,
with all hbase table major compact thus all data is local.

       I have hoped that the short circuit read will improve the
performance.


       While the test result is that with short circuit read enabled, the
performance actually dropped 10-15%. Say scan a 50G table cost around 100s
instead of 90s.


       My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu




--
Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Arpit Gupta

Yes,  this way also confirms that short circuit read is enabled on my cluster.

13/02/16 14:07:34 DEBUG hdfs.DFSClient: Short circuit read is true

13/02/16 14:07:34 DEBUG hdfs.DFSClient: New BlockReaderLocal for file /mnt/DP_disk4/raymond/hdfs/data/current/subdir63/blk_-2736548898990727638 of size 134217728 startOffset 0 length 134217728 short circuit checksum false

So , any possibility that other setting might impact short circuit read to has worse performance than read through datanode?

Raymond



>Another way to check if short circuit read is configured correctly.

>As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

>export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

>On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

>This would confirm that short circuit read is happening.

--
>Arpit Gupta
>Hortonworks Inc.
>http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:


Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.



If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
could result in performance penalty and some warn-logging, as local reads will
be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
wrote:

Hi

       I tried to use short circuit read to improve my hbase cluster MR
scan performance.


       I have the following setting in hdfs-site.xml

       dfs.client.read.shortcircuit set to true
       dfs.block.local-path-access.user set to MR job runner.

       The cluster is 1+4 node and each data node have 16cpu/4HDD,
with all hbase table major compact thus all data is local.

       I have hoped that the short circuit read will improve the
performance.


       While the test result is that with short circuit read enabled, the
performance actually dropped 10-15%. Say scan a 50G table cost around 100s
instead of 90s.


       My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu




--
Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Arpit Gupta <ar...@hortonworks.com>.
Another way to check if short circuit read is configured correctly.

As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

This would confirm that short circuit read is happening.

--
Arpit Gupta
Hortonworks Inc.
http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.
> 
>> 
>> If you want HBase to leverage the shortcircuit, the DN config
>> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
>> hbase, for example), and the hbase-site.xml should have
>> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
>> could result in performance penalty and some warn-logging, as local reads will
>> be attempted but will begin to fail.
>> 
>> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
>> wrote:
>>> Hi
>>> 
>>>        I tried to use short circuit read to improve my hbase cluster MR
>> scan performance.
>>> 
>>>        I have the following setting in hdfs-site.xml
>>> 
>>>        dfs.client.read.shortcircuit set to true
>>>        dfs.block.local-path-access.user set to MR job runner.
>>> 
>>>        The cluster is 1+4 node and each data node have 16cpu/4HDD,
>> with all hbase table major compact thus all data is local.
>>>        I have hoped that the short circuit read will improve the
>> performance.
>>> 
>>>        While the test result is that with short circuit read enabled, the
>> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
>> instead of 90s.
>>> 
>>>        My hadoop version is 1.1.1, any idea on this? Thx!
>>> 
>>> Best Regards,
>>> Raymond Liu
>>> 
>>> 
>> 
>> 
>> 
>> --
>> Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Arpit Gupta <ar...@hortonworks.com>.
Another way to check if short circuit read is configured correctly.

As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

This would confirm that short circuit read is happening.

--
Arpit Gupta
Hortonworks Inc.
http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.
> 
>> 
>> If you want HBase to leverage the shortcircuit, the DN config
>> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
>> hbase, for example), and the hbase-site.xml should have
>> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
>> could result in performance penalty and some warn-logging, as local reads will
>> be attempted but will begin to fail.
>> 
>> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
>> wrote:
>>> Hi
>>> 
>>>        I tried to use short circuit read to improve my hbase cluster MR
>> scan performance.
>>> 
>>>        I have the following setting in hdfs-site.xml
>>> 
>>>        dfs.client.read.shortcircuit set to true
>>>        dfs.block.local-path-access.user set to MR job runner.
>>> 
>>>        The cluster is 1+4 node and each data node have 16cpu/4HDD,
>> with all hbase table major compact thus all data is local.
>>>        I have hoped that the short circuit read will improve the
>> performance.
>>> 
>>>        While the test result is that with short circuit read enabled, the
>> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
>> instead of 90s.
>>> 
>>>        My hadoop version is 1.1.1, any idea on this? Thx!
>>> 
>>> Best Regards,
>>> Raymond Liu
>>> 
>>> 
>> 
>> 
>> 
>> --
>> Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Arpit Gupta <ar...@hortonworks.com>.
Another way to check if short circuit read is configured correctly.

As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

This would confirm that short circuit read is happening.

--
Arpit Gupta
Hortonworks Inc.
http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.
> 
>> 
>> If you want HBase to leverage the shortcircuit, the DN config
>> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
>> hbase, for example), and the hbase-site.xml should have
>> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
>> could result in performance penalty and some warn-logging, as local reads will
>> be attempted but will begin to fail.
>> 
>> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
>> wrote:
>>> Hi
>>> 
>>>        I tried to use short circuit read to improve my hbase cluster MR
>> scan performance.
>>> 
>>>        I have the following setting in hdfs-site.xml
>>> 
>>>        dfs.client.read.shortcircuit set to true
>>>        dfs.block.local-path-access.user set to MR job runner.
>>> 
>>>        The cluster is 1+4 node and each data node have 16cpu/4HDD,
>> with all hbase table major compact thus all data is local.
>>>        I have hoped that the short circuit read will improve the
>> performance.
>>> 
>>>        While the test result is that with short circuit read enabled, the
>> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
>> instead of 90s.
>>> 
>>>        My hadoop version is 1.1.1, any idea on this? Thx!
>>> 
>>> Best Regards,
>>> Raymond Liu
>>> 
>>> 
>> 
>> 
>> 
>> --
>> Harsh J


Re: why my test result on dfs short circuit read is slower?

Posted by Arpit Gupta <ar...@hortonworks.com>.
Another way to check if short circuit read is configured correctly.

As the user who is configured for short circuit read issue the following command on a node where you expect the data to be read locally.

export HADOOP_ROOT_LOGGER=debug,console; hadoop dfs -cat /path/to/file_on_hdfs

On the console you should see something like "hdfs.DFSClient: New BlockReaderLocal for file...."

This would confirm that short circuit read is happening.

--
Arpit Gupta
Hortonworks Inc.
http://hortonworks.com/

On Feb 15, 2013, at 9:53 PM, "Liu, Raymond" <ra...@intel.com> wrote:

> Hi Harsh
> 
> Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.
> 
> And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.
> 
>> 
>> If you want HBase to leverage the shortcircuit, the DN config
>> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
>> hbase, for example), and the hbase-site.xml should have
>> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
>> could result in performance penalty and some warn-logging, as local reads will
>> be attempted but will begin to fail.
>> 
>> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
>> wrote:
>>> Hi
>>> 
>>>        I tried to use short circuit read to improve my hbase cluster MR
>> scan performance.
>>> 
>>>        I have the following setting in hdfs-site.xml
>>> 
>>>        dfs.client.read.shortcircuit set to true
>>>        dfs.block.local-path-access.user set to MR job runner.
>>> 
>>>        The cluster is 1+4 node and each data node have 16cpu/4HDD,
>> with all hbase table major compact thus all data is local.
>>>        I have hoped that the short circuit read will improve the
>> performance.
>>> 
>>>        While the test result is that with short circuit read enabled, the
>> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
>> instead of 90s.
>>> 
>>>        My hadoop version is 1.1.1, any idea on this? Thx!
>>> 
>>> Best Regards,
>>> Raymond Liu
>>> 
>>> 
>> 
>> 
>> 
>> --
>> Harsh J


RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.

> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> > Hi
> >
> >         I tried to use short circuit read to improve my hbase cluster MR
> scan performance.
> >
> >         I have the following setting in hdfs-site.xml
> >
> >         dfs.client.read.shortcircuit set to true
> >         dfs.block.local-path-access.user set to MR job runner.
> >
> >         The cluster is 1+4 node and each data node have 16cpu/4HDD,
> with all hbase table major compact thus all data is local.
> >         I have hoped that the short circuit read will improve the
> performance.
> >
> >         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> >
> >         My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> 
> 
> 
> --
> Harsh J

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.

> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> > Hi
> >
> >         I tried to use short circuit read to improve my hbase cluster MR
> scan performance.
> >
> >         I have the following setting in hdfs-site.xml
> >
> >         dfs.client.read.shortcircuit set to true
> >         dfs.block.local-path-access.user set to MR job runner.
> >
> >         The cluster is 1+4 node and each data node have 16cpu/4HDD,
> with all hbase table major compact thus all data is local.
> >         I have hoped that the short circuit read will improve the
> performance.
> >
> >         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> >
> >         My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> 
> 
> 
> --
> Harsh J

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.

> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> > Hi
> >
> >         I tried to use short circuit read to improve my hbase cluster MR
> scan performance.
> >
> >         I have the following setting in hdfs-site.xml
> >
> >         dfs.client.read.shortcircuit set to true
> >         dfs.block.local-path-access.user set to MR job runner.
> >
> >         The cluster is 1+4 node and each data node have 16cpu/4HDD,
> with all hbase table major compact thus all data is local.
> >         I have hoped that the short circuit read will improve the
> performance.
> >
> >         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> >
> >         My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> 
> 
> 
> --
> Harsh J

RE: why my test result on dfs short circuit read is slower?

Posted by "Liu, Raymond" <ra...@intel.com>.
Hi Harsh

Yes, I did set both of these. While not in hbase-site.xml but hdfs-site.xml.

And I have double confirmed that local reads are performed, since there are no Error in datanode logs, and by watching lo network IO.

> 
> If you want HBase to leverage the shortcircuit, the DN config
> "dfs.block.local-path-access.user" should be set to the user running HBase (i.e.
> hbase, for example), and the hbase-site.xml should have
> "dfs.client.read.shortcircuit" defined in all its RegionServers. Doing this wrong
> could result in performance penalty and some warn-logging, as local reads will
> be attempted but will begin to fail.
> 
> On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com>
> wrote:
> > Hi
> >
> >         I tried to use short circuit read to improve my hbase cluster MR
> scan performance.
> >
> >         I have the following setting in hdfs-site.xml
> >
> >         dfs.client.read.shortcircuit set to true
> >         dfs.block.local-path-access.user set to MR job runner.
> >
> >         The cluster is 1+4 node and each data node have 16cpu/4HDD,
> with all hbase table major compact thus all data is local.
> >         I have hoped that the short circuit read will improve the
> performance.
> >
> >         While the test result is that with short circuit read enabled, the
> performance actually dropped 10-15%. Say scan a 50G table cost around 100s
> instead of 90s.
> >
> >         My hadoop version is 1.1.1, any idea on this? Thx!
> >
> > Best Regards,
> > Raymond Liu
> >
> >
> 
> 
> 
> --
> Harsh J

Re: why my test result on dfs short circuit read is slower?

Posted by Harsh J <ha...@cloudera.com>.
If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running
HBase (i.e. hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
this wrong could result in performance penalty and some warn-logging,
as local reads will be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com> wrote:
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the performance.
>
>         While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu
>
>



--
Harsh J

Re: why my test result on dfs short circuit read is slower?

Posted by Harsh J <ha...@cloudera.com>.
If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running
HBase (i.e. hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
this wrong could result in performance penalty and some warn-logging,
as local reads will be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com> wrote:
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the performance.
>
>         While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu
>
>



--
Harsh J

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Hi Raymond,

did you enable security feature in your cluster?  there'll be no obvious benefit be found if so.

Regards,
Liang
_______________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:10
收件人: user@hadoop.apache.org
主题: why my test result on dfs short circuit read is slower?

Hi

        I tried to use short circuit read to improve my hbase cluster MR scan performance.

        I have the following setting in hdfs-site.xml

        dfs.client.read.shortcircuit set to true
        dfs.block.local-path-access.user set to MR job runner.

        The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
        I have hoped that the short circuit read will improve the performance.

        While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.

        My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu

Re: why my test result on dfs short circuit read is slower?

Posted by Harsh J <ha...@cloudera.com>.
If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running
HBase (i.e. hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
this wrong could result in performance penalty and some warn-logging,
as local reads will be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com> wrote:
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the performance.
>
>         While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu
>
>



--
Harsh J

Re: why my test result on dfs short circuit read is slower?

Posted by Harsh J <ha...@cloudera.com>.
If you want HBase to leverage the shortcircuit, the DN config
"dfs.block.local-path-access.user" should be set to the user running
HBase (i.e. hbase, for example), and the hbase-site.xml should have
"dfs.client.read.shortcircuit" defined in all its RegionServers. Doing
this wrong could result in performance penalty and some warn-logging,
as local reads will be attempted but will begin to fail.

On Sat, Feb 16, 2013 at 8:40 AM, Liu, Raymond <ra...@intel.com> wrote:
> Hi
>
>         I tried to use short circuit read to improve my hbase cluster MR scan performance.
>
>         I have the following setting in hdfs-site.xml
>
>         dfs.client.read.shortcircuit set to true
>         dfs.block.local-path-access.user set to MR job runner.
>
>         The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
>         I have hoped that the short circuit read will improve the performance.
>
>         While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.
>
>         My hadoop version is 1.1.1, any idea on this? Thx!
>
> Best Regards,
> Raymond Liu
>
>



--
Harsh J

答复: why my test result on dfs short circuit read is slower?

Posted by 谢良 <xi...@xiaomi.com>.
Hi Raymond,

did you enable security feature in your cluster?  there'll be no obvious benefit be found if so.

Regards,
Liang
_______________________________________
发件人: Liu, Raymond [raymond.liu@intel.com]
发送时间: 2013年2月16日 11:10
收件人: user@hadoop.apache.org
主题: why my test result on dfs short circuit read is slower?

Hi

        I tried to use short circuit read to improve my hbase cluster MR scan performance.

        I have the following setting in hdfs-site.xml

        dfs.client.read.shortcircuit set to true
        dfs.block.local-path-access.user set to MR job runner.

        The cluster is 1+4 node and each data node have 16cpu/4HDD, with all hbase table major compact thus all data is local.
        I have hoped that the short circuit read will improve the performance.

        While the test result is that with short circuit read enabled, the performance actually dropped 10-15%. Say scan a 50G table cost around 100s instead of 90s.

        My hadoop version is 1.1.1, any idea on this? Thx!

Best Regards,
Raymond Liu