HDFSCommands.apt.vm 22 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408
  1. ~~ Licensed under the Apache License, Version 2.0 (the "License");
  2. ~~ you may not use this file except in compliance with the License.
  3. ~~ You may obtain a copy of the License at
  4. ~~
  5. ~~ http://www.apache.org/licenses/LICENSE-2.0
  6. ~~
  7. ~~ Unless required by applicable law or agreed to in writing, software
  8. ~~ distributed under the License is distributed on an "AS IS" BASIS,
  9. ~~ WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  10. ~~ See the License for the specific language governing permissions and
  11. ~~ limitations under the License. See accompanying LICENSE file.
  12. ---
  13. HDFS Commands Guide
  14. ---
  15. ---
  16. ${maven.build.timestamp}
  17. HDFS Commands Guide
  18. %{toc|section=1|fromDepth=2|toDepth=4}
  19. * Overview
  20. All HDFS commands are invoked by the <<<bin/hdfs>>> script. Running the
  21. hdfs script without any arguments prints the description for all
  22. commands.
  23. Usage: <<<hdfs [--config confdir] [COMMAND] [GENERIC_OPTIONS]
  24. [COMMAND_OPTIONS]>>>
  25. Hadoop has an option parsing framework that employs parsing generic options
  26. as well as running classes.
  27. *-----------------------+---------------+
  28. || COMMAND_OPTION || Description
  29. *-----------------------+---------------+
  30. | <<<--config confdir>>>| Overwrites the default Configuration directory.
  31. | | Default is <<<${HADOOP_HOME}/conf>>>.
  32. *-----------------------+---------------+
  33. | GENERIC_OPTIONS | The common set of options supported by multiple
  34. | | commands. Full list is
  35. | | {{{../hadoop-common/CommandsManual.html#Generic_Options}here}}.
  36. *-----------------------+---------------+
  37. | COMMAND_OPTIONS | Various commands with their options are described in
  38. | | the following sections. The commands have been
  39. | | grouped into {{{User Commands}}} and
  40. | | {{{Administration Commands}}}.
  41. *-----------------------+---------------+
  42. * User Commands
  43. Commands useful for users of a hadoop cluster.
  44. ** <<<dfs>>>
  45. Usage: <<<hdfs dfs [GENERIC_OPTIONS] [COMMAND_OPTIONS]>>>
  46. Run a filesystem command on the file system supported in Hadoop.
  47. The various COMMAND_OPTIONS can be found at
  48. {{{../hadoop-common/FileSystemShell.html}File System Shell Guide}}.
  49. ** <<<fetchdt>>>
  50. Gets Delegation Token from a NameNode.
  51. See {{{./HdfsUserGuide.html#fetchdt}fetchdt}} for more info.
  52. Usage: <<<hdfs fetchdt [GENERIC_OPTIONS]
  53. [--webservice <namenode_http_addr>] <path> >>>
  54. *------------------------------+---------------------------------------------+
  55. || COMMAND_OPTION || Description
  56. *------------------------------+---------------------------------------------+
  57. | <fileName> | File name to store the token into.
  58. *------------------------------+---------------------------------------------+
  59. | --webservice <https_address> | use http protocol instead of RPC
  60. *------------------------------+---------------------------------------------+
  61. ** <<<fsck>>>
  62. Runs a HDFS filesystem checking utility.
  63. See {{{./HdfsUserGuide.html#fsck}fsck}} for more info.
  64. Usage: <<<hdfs fsck [GENERIC_OPTIONS] <path>
  65. [-move | -delete | -openforwrite]
  66. [-files [-blocks [-locations | -racks]]]
  67. [-showprogress]>>>
  68. *------------------+---------------------------------------------+
  69. || COMMAND_OPTION || Description
  70. *------------------+---------------------------------------------+
  71. | <path> | Start checking from this path.
  72. *------------------+---------------------------------------------+
  73. | -move | Move corrupted files to /lost+found
  74. *------------------+---------------------------------------------+
  75. | -delete | Delete corrupted files.
  76. *------------------+---------------------------------------------+
  77. | -openforwrite | Print out files opened for write.
  78. *------------------+---------------------------------------------+
  79. | -files | Print out files being checked.
  80. *------------------+---------------------------------------------+
  81. | -blocks | Print out block report.
  82. *------------------+---------------------------------------------+
  83. | -locations | Print out locations for every block.
  84. *------------------+---------------------------------------------+
  85. | -racks | Print out network topology for data-node locations.
  86. *------------------+---------------------------------------------+
  87. | -showprogress | Print out dots for progress in output. Default is OFF
  88. | | (no progress).
  89. *------------------+---------------------------------------------+
  90. * Administration Commands
  91. Commands useful for administrators of a hadoop cluster.
  92. ** <<<balancer>>>
  93. Runs a cluster balancing utility. An administrator can simply press Ctrl-C
  94. to stop the rebalancing process. See
  95. {{{./HdfsUserGuide.html#Balancer}Balancer}} for more details.
  96. Usage: <<<hdfs balancer [-threshold <threshold>] [-policy <policy>]>>>
  97. *------------------------+----------------------------------------------------+
  98. || COMMAND_OPTION | Description
  99. *------------------------+----------------------------------------------------+
  100. | -threshold <threshold> | Percentage of disk capacity. This overwrites the
  101. | | default threshold.
  102. *------------------------+----------------------------------------------------+
  103. | -policy <policy> | <<<datanode>>> (default): Cluster is balanced if
  104. | | each datanode is balanced. \
  105. | | <<<blockpool>>>: Cluster is balanced if each block
  106. | | pool in each datanode is balanced.
  107. *------------------------+----------------------------------------------------+
  108. Note that the <<<blockpool>>> policy is more strict than the <<<datanode>>>
  109. policy.
  110. ** <<<datanode>>>
  111. Runs a HDFS datanode.
  112. Usage: <<<hdfs datanode [-regular | -rollback | -rollingupgrace rollback]>>>
  113. *-----------------+-----------------------------------------------------------+
  114. || COMMAND_OPTION || Description
  115. *-----------------+-----------------------------------------------------------+
  116. | -regular | Normal datanode startup (default).
  117. *-----------------+-----------------------------------------------------------+
  118. | -rollback | Rollsback the datanode to the previous version. This should
  119. | | be used after stopping the datanode and distributing the
  120. | | old hadoop version.
  121. *-----------------+-----------------------------------------------------------+
  122. | -rollingupgrade rollback | Rollsback a rolling upgrade operation.
  123. *-----------------+-----------------------------------------------------------+
  124. ** <<<dfsadmin>>>
  125. Runs a HDFS dfsadmin client.
  126. Usage: <<<hdfs dfsadmin [GENERIC_OPTIONS]
  127. [-report [-live] [-dead] [-decommissioning]]
  128. [-safemode enter | leave | get | wait]
  129. [-saveNamespace]
  130. [-rollEdits]
  131. [-restoreFailedStorage true|false|check]
  132. [-refreshNodes]
  133. [-setQuota <quota> <dirname>...<dirname>]
  134. [-clrQuota <dirname>...<dirname>]
  135. [-setSpaceQuota <quota> <dirname>...<dirname>]
  136. [-clrSpaceQuota <dirname>...<dirname>]
  137. [-finalizeUpgrade]
  138. [-rollingUpgrade [<query>|<prepare>|<finalize>]]
  139. [-metasave filename]
  140. [-refreshServiceAcl]
  141. [-refreshUserToGroupsMappings]
  142. [-refreshSuperUserGroupsConfiguration]
  143. [-refreshCallQueue]
  144. [-refresh <host:ipc_port> <key> [arg1..argn]]
  145. [-printTopology]
  146. [-refreshNamenodes datanodehost:port]
  147. [-deleteBlockPool datanode-host:port blockpoolId [force]]
  148. [-setBalancerBandwidth <bandwidth in bytes per second>]
  149. [-allowSnapshot <snapshotDir>]
  150. [-disallowSnapshot <snapshotDir>]
  151. [-fetchImage <local directory>]
  152. [-shutdownDatanode <datanode_host:ipc_port> [upgrade]]
  153. [-getDatanodeInfo <datanode_host:ipc_port>]
  154. [-help [cmd]]>>>
  155. *-----------------+-----------------------------------------------------------+
  156. || COMMAND_OPTION || Description
  157. *-----------------+-----------------------------------------------------------+
  158. | -report [-live] [-dead] [-decommissioning] | Reports basic filesystem
  159. | information and statistics. Optional flags may be used to
  160. | filter the list of displayed DataNodes.
  161. *-----------------+-----------------------------------------------------------+
  162. | -safemode enter\|leave\|get\|wait | Safe mode maintenance command. Safe
  163. | mode is a Namenode state in which it \
  164. | 1. does not accept changes to the name space (read-only) \
  165. | 2. does not replicate or delete blocks. \
  166. | Safe mode is entered automatically at Namenode startup, and
  167. | leaves safe mode automatically when the configured minimum
  168. | percentage of blocks satisfies the minimum replication
  169. | condition. Safe mode can also be entered manually, but then
  170. | it can only be turned off manually as well.
  171. *-----------------+-----------------------------------------------------------+
  172. | -saveNamespace | Save current namespace into storage directories and reset
  173. | edits log. Requires safe mode.
  174. *-----------------+-----------------------------------------------------------+
  175. | -rollEdits | Rolls the edit log on the active NameNode.
  176. *-----------------+-----------------------------------------------------------+
  177. | -restoreFailedStorage true\|false\|check | This option will turn on/off
  178. | automatic attempt to restore failed storage replicas.
  179. | If a failed storage becomes available again the system will
  180. | attempt to restore edits and/or fsimage during checkpoint.
  181. | 'check' option will return current setting.
  182. *-----------------+-----------------------------------------------------------+
  183. | -refreshNodes | Re-read the hosts and exclude files to update the set of
  184. | Datanodes that are allowed to connect to the Namenode and
  185. | those that should be decommissioned or recommissioned.
  186. *-----------------+-----------------------------------------------------------+
  187. | -setQuota \<quota\> \<dirname\>...\<dirname\> | See
  188. | {{{../hadoop-hdfs/HdfsQuotaAdminGuide.html#Administrative_Commands}HDFS Quotas Guide}}
  189. | for the detail.
  190. *-----------------+-----------------------------------------------------------+
  191. | -clrQuota \<dirname\>...\<dirname\> | See
  192. | {{{../hadoop-hdfs/HdfsQuotaAdminGuide.html#Administrative_Commands}HDFS Quotas Guide}}
  193. | for the detail.
  194. *-----------------+-----------------------------------------------------------+
  195. | -setSpaceQuota \<quota\> \<dirname\>...\<dirname\> | See
  196. | {{{../hadoop-hdfs/HdfsQuotaAdminGuide.html#Administrative_Commands}HDFS Quotas Guide}}
  197. | for the detail.
  198. *-----------------+-----------------------------------------------------------+
  199. | -clrSpaceQuota \<dirname\>...\<dirname\> | See
  200. | {{{../hadoop-hdfs/HdfsQuotaAdminGuide.html#Administrative_Commands}HDFS Quotas Guide}}
  201. | for the detail.
  202. *-----------------+-----------------------------------------------------------+
  203. | -finalizeUpgrade| Finalize upgrade of HDFS. Datanodes delete their previous
  204. | version working directories, followed by Namenode doing the
  205. | same. This completes the upgrade process.
  206. *-----------------+-----------------------------------------------------------+
  207. | -rollingUpgrade [\<query\>\|\<prepare\>\|\<finalize\>] | See
  208. | {{{../hadoop-hdfs/HdfsRollingUpgrade.html#dfsadmin_-rollingUpgrade}Rolling Upgrade document}}
  209. | for the detail.
  210. *-----------------+-----------------------------------------------------------+
  211. | -metasave filename | Save Namenode's primary data structures to <filename> in
  212. | the directory specified by hadoop.log.dir property.
  213. | <filename> is overwritten if it exists.
  214. | <filename> will contain one line for each of the following\
  215. | 1. Datanodes heart beating with Namenode\
  216. | 2. Blocks waiting to be replicated\
  217. | 3. Blocks currrently being replicated\
  218. | 4. Blocks waiting to be deleted
  219. *-----------------+-----------------------------------------------------------+
  220. | -refreshServiceAcl | Reload the service-level authorization policy file.
  221. *-----------------+-----------------------------------------------------------+
  222. | -refreshUserToGroupsMappings | Refresh user-to-groups mappings.
  223. *-----------------+-----------------------------------------------------------+
  224. | -refreshSuperUserGroupsConfiguration |Refresh superuser proxy groups mappings
  225. *-----------------+-----------------------------------------------------------+
  226. | -refreshCallQueue | Reload the call queue from config.
  227. *-----------------+-----------------------------------------------------------+
  228. | -refresh \<host:ipc_port\> \<key\> [arg1..argn] | Triggers a runtime-refresh
  229. | of the resource specified by \<key\> on \<host:ipc_port\>.
  230. | All other args after are sent to the host.
  231. *-----------------+-----------------------------------------------------------+
  232. | -printTopology | Print a tree of the racks and their nodes as reported by
  233. | the Namenode
  234. *-----------------+-----------------------------------------------------------+
  235. | -refreshNamenodes datanodehost:port | For the given datanode, reloads the
  236. | configuration files, stops serving the removed block-pools
  237. | and starts serving new block-pools.
  238. *-----------------+-----------------------------------------------------------+
  239. | -deleteBlockPool datanode-host:port blockpoolId [force] | If force is passed,
  240. | block pool directory for the given blockpool id on the
  241. | given datanode is deleted along with its contents,
  242. | otherwise the directory is deleted only if it is empty.
  243. | The command will fail if datanode is still serving the
  244. | block pool. Refer to refreshNamenodes to shutdown a block
  245. | pool service on a datanode.
  246. *-----------------+-----------------------------------------------------------+
  247. | -setBalancerBandwidth \<bandwidth in bytes per second\> | Changes the network
  248. | bandwidth used by each datanode during HDFS block
  249. | balancing. \<bandwidth\> is the maximum number of bytes per
  250. | second that will be used by each datanode. This value
  251. | overrides the dfs.balance.bandwidthPerSec parameter.\
  252. | NOTE: The new value is not persistent on the DataNode.
  253. *-----------------+-----------------------------------------------------------+
  254. | -allowSnapshot \<snapshotDir\> | Allowing snapshots of a directory to be
  255. | created. If the operation completes successfully, the
  256. | directory becomes snapshottable.
  257. *-----------------+-----------------------------------------------------------+
  258. | -disallowSnapshot \<snapshotDir\> | Disallowing snapshots of a directory to
  259. | be created. All snapshots of the directory must be deleted
  260. | before disallowing snapshots.
  261. *-----------------+-----------------------------------------------------------+
  262. | -fetchImage \<local directory\> | Downloads the most recent fsimage from the
  263. | NameNode and saves it in the specified local directory.
  264. *-----------------+-----------------------------------------------------------+
  265. | -shutdownDatanode \<datanode_host:ipc_port\> [upgrade] | Submit a shutdown
  266. | request for the given datanode. See
  267. | {{{./HdfsRollingUpgrade.html#dfsadmin_-shutdownDatanode}Rolling Upgrade document}}
  268. | for the detail.
  269. *-----------------+-----------------------------------------------------------+
  270. | -getDatanodeInfo \<datanode_host:ipc_port\> | Get the information about the
  271. | given datanode. See
  272. | {{{./HdfsRollingUpgrade.html#dfsadmin_-getDatanodeInfo}Rolling Upgrade document}}
  273. | for the detail.
  274. *-----------------+-----------------------------------------------------------+
  275. | -help [cmd] | Displays help for the given command or all commands if none
  276. | is specified.
  277. *-----------------+-----------------------------------------------------------+
  278. ** <<<namenode>>>
  279. Runs the namenode. More info about the upgrade, rollback and finalize is at
  280. {{{./HdfsUserGuide.html#Upgrade_and_Rollback}Upgrade Rollback}}.
  281. Usage: <<<hdfs namenode [-backup] |
  282. [-checkpoint] |
  283. [-format [-clusterid cid ] [-force] [-nonInteractive] ] |
  284. [-upgrade [-clusterid cid] [-renameReserved<k-v pairs>] ] |
  285. [-upgradeOnly [-clusterid cid] [-renameReserved<k-v pairs>] ] |
  286. [-rollback] |
  287. [-rollingUpgrade <downgrade|rollback> ] |
  288. [-finalize] |
  289. [-importCheckpoint] |
  290. [-initializeSharedEdits] |
  291. [-bootstrapStandby] |
  292. [-recover [-force] ] |
  293. [-metadataVersion ]>>>
  294. *--------------------+--------------------------------------------------------+
  295. || COMMAND_OPTION || Description
  296. *--------------------+--------------------------------------------------------+
  297. | -backup | Start backup node.
  298. *--------------------+--------------------------------------------------------+
  299. | -checkpoint | Start checkpoint node.
  300. *--------------------+--------------------------------------------------------+
  301. | -format [-clusterid cid] [-force] [-nonInteractive] | Formats the specified
  302. | NameNode. It starts the NameNode, formats it and then
  303. | shut it down. -force option formats if the name
  304. | directory exists. -nonInteractive option aborts if the
  305. | name directory exists, unless -force option is specified.
  306. *--------------------+--------------------------------------------------------+
  307. | -upgrade [-clusterid cid] [-renameReserved\<k-v pairs\>] | Namenode should be
  308. | started with upgrade option after
  309. | the distribution of new Hadoop version.
  310. *--------------------+--------------------------------------------------------+
  311. | -upgradeOnly [-clusterid cid] [-renameReserved\<k-v pairs\>] | Upgrade the
  312. | specified NameNode and then shutdown it.
  313. *--------------------+--------------------------------------------------------+
  314. | -rollback | Rollsback the NameNode to the previous version. This
  315. | should be used after stopping the cluster and
  316. | distributing the old Hadoop version.
  317. *--------------------+--------------------------------------------------------+
  318. | -rollingUpgrade \<downgrade\|rollback\|started\> | See
  319. | {{{./HdfsRollingUpgrade.html#NameNode_Startup_Options}Rolling Upgrade document}}
  320. | for the detail.
  321. *--------------------+--------------------------------------------------------+
  322. | -finalize | Finalize will remove the previous state of the files
  323. | system. Recent upgrade will become permanent. Rollback
  324. | option will not be available anymore. After finalization
  325. | it shuts the NameNode down.
  326. *--------------------+--------------------------------------------------------+
  327. | -importCheckpoint | Loads image from a checkpoint directory and save it
  328. | into the current one. Checkpoint dir is read from
  329. | property fs.checkpoint.dir
  330. *--------------------+--------------------------------------------------------+
  331. | -initializeSharedEdits | Format a new shared edits dir and copy in enough
  332. | edit log segments so that the standby NameNode can start
  333. | up.
  334. *--------------------+--------------------------------------------------------+
  335. | -bootstrapStandby | Allows the standby NameNode's storage directories to be
  336. | bootstrapped by copying the latest namespace snapshot
  337. | from the active NameNode. This is used when first
  338. | configuring an HA cluster.
  339. *--------------------+--------------------------------------------------------+
  340. | -recover [-force] | Recover lost metadata on a corrupt filesystem. See
  341. | {{{./HdfsUserGuide.html#Recovery_Mode}HDFS User Guide}}
  342. | for the detail.
  343. *--------------------+--------------------------------------------------------+
  344. | -metadataVersion | Verify that configured directories exist, then print the
  345. | metadata versions of the software and the image.
  346. *--------------------+--------------------------------------------------------+
  347. ** <<<secondarynamenode>>>
  348. Runs the HDFS secondary namenode.
  349. See {{{./HdfsUserGuide.html#Secondary_NameNode}Secondary Namenode}}
  350. for more info.
  351. Usage: <<<hdfs secondarynamenode [-checkpoint [force]] | [-format] |
  352. [-geteditsize]>>>
  353. *----------------------+------------------------------------------------------+
  354. || COMMAND_OPTION || Description
  355. *----------------------+------------------------------------------------------+
  356. | -checkpoint [force] | Checkpoints the SecondaryNameNode if EditLog size
  357. | >= fs.checkpoint.size. If <<<force>>> is used,
  358. | checkpoint irrespective of EditLog size.
  359. *----------------------+------------------------------------------------------+
  360. | -format | Format the local storage during startup.
  361. *----------------------+------------------------------------------------------+
  362. | -geteditsize | Prints the number of uncheckpointed transactions on
  363. | the NameNode.
  364. *----------------------+------------------------------------------------------+