namenode_operations.cc 20 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635
  1. /**
  2. * Licensed to the Apache Software Foundation (ASF) under one
  3. * or more contributor license agreements. See the NOTICE file
  4. * distributed with this work for additional information
  5. * regarding copyright ownership. The ASF licenses this file
  6. * to you under the Apache License, Version 2.0 (the
  7. * "License"); you may not use this file except in compliance
  8. * with the License. You may obtain a copy of the License at
  9. *
  10. * http://www.apache.org/licenses/LICENSE-2.0
  11. *
  12. * Unless required by applicable law or agreed to in writing, software
  13. * distributed under the License is distributed on an "AS IS" BASIS,
  14. * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  15. * See the License for the specific language governing permissions and
  16. * limitations under the License.
  17. */
  18. #include "filesystem.h"
  19. #include "common/continuation/asio.h"
  20. #include <asio/ip/tcp.hpp>
  21. #include <functional>
  22. #include <limits>
  23. #include <future>
  24. #include <tuple>
  25. #include <iostream>
  26. #include <pwd.h>
  27. #include <utility>
  28. #define FMT_THIS_ADDR "this=" << (void*)this
  29. using ::asio::ip::tcp;
  30. namespace hdfs {
  31. /*****************************************************************************
  32. * NAMENODE OPERATIONS
  33. ****************************************************************************/
  34. void NameNodeOperations::Connect(const std::string &cluster_name,
  35. const std::vector<ResolvedNamenodeInfo> &servers,
  36. std::function<void(const Status &)> &&handler) {
  37. engine_.Connect(cluster_name, servers, handler);
  38. }
  39. void NameNodeOperations::GetBlockLocations(const std::string & path, uint64_t offset, uint64_t length,
  40. std::function<void(const Status &, std::shared_ptr<const struct FileInfo>)> handler)
  41. {
  42. using ::hadoop::hdfs::GetBlockLocationsRequestProto;
  43. using ::hadoop::hdfs::GetBlockLocationsResponseProto;
  44. LOG_TRACE(kFileSystem, << "NameNodeOperations::GetBlockLocations("
  45. << FMT_THIS_ADDR << ", path=" << path << ", ...) called");
  46. if (path.empty()) {
  47. handler(Status::InvalidArgument("GetBlockLocations: argument 'path' cannot be empty"), nullptr);
  48. return;
  49. }
  50. //Protobuf gives an error 'Negative value is not supported'
  51. //if the high bit is set in uint64 in GetBlockLocations
  52. if (IsHighBitSet(offset)) {
  53. handler(Status::InvalidArgument("GetBlockLocations: argument 'offset' cannot have high bit set"), nullptr);
  54. return;
  55. }
  56. if (IsHighBitSet(length)) {
  57. handler(Status::InvalidArgument("GetBlockLocations: argument 'length' cannot have high bit set"), nullptr);
  58. return;
  59. }
  60. GetBlockLocationsRequestProto req;
  61. req.set_src(path);
  62. req.set_offset(offset);
  63. req.set_length(length);
  64. auto resp = std::make_shared<GetBlockLocationsResponseProto>();
  65. namenode_.GetBlockLocations(&req, resp, [resp, handler](const Status &stat) {
  66. if (stat.ok()) {
  67. auto file_info = std::make_shared<struct FileInfo>();
  68. auto locations = resp->locations();
  69. file_info->file_length_ = locations.filelength();
  70. file_info->last_block_complete_ = locations.islastblockcomplete();
  71. file_info->under_construction_ = locations.underconstruction();
  72. for (const auto &block : locations.blocks()) {
  73. file_info->blocks_.push_back(block);
  74. }
  75. if (!locations.islastblockcomplete() &&
  76. locations.has_lastblock() && locations.lastblock().b().numbytes()) {
  77. file_info->blocks_.push_back(locations.lastblock());
  78. file_info->file_length_ += locations.lastblock().b().numbytes();
  79. }
  80. handler(stat, file_info);
  81. } else {
  82. handler(stat, nullptr);
  83. }
  84. });
  85. }
  86. void NameNodeOperations::GetPreferredBlockSize(const std::string & path,
  87. std::function<void(const Status &, const uint64_t)> handler)
  88. {
  89. using ::hadoop::hdfs::GetPreferredBlockSizeRequestProto;
  90. using ::hadoop::hdfs::GetPreferredBlockSizeResponseProto;
  91. LOG_TRACE(kFileSystem, << "NameNodeOperations::GetPreferredBlockSize("
  92. << FMT_THIS_ADDR << ", path=" << path << ") called");
  93. if (path.empty()) {
  94. handler(Status::InvalidArgument("GetPreferredBlockSize: argument 'path' cannot be empty"), -1);
  95. return;
  96. }
  97. GetPreferredBlockSizeRequestProto req;
  98. req.set_filename(path);
  99. auto resp = std::make_shared<GetPreferredBlockSizeResponseProto>();
  100. namenode_.GetPreferredBlockSize(&req, resp, [resp, handler, path](const Status &stat) {
  101. if (stat.ok() && resp -> has_bsize()) {
  102. uint64_t block_size = resp -> bsize();
  103. handler(stat, block_size);
  104. } else {
  105. handler(stat, -1);
  106. }
  107. });
  108. }
  109. void NameNodeOperations::SetReplication(const std::string & path, int16_t replication,
  110. std::function<void(const Status &)> handler)
  111. {
  112. using ::hadoop::hdfs::SetReplicationRequestProto;
  113. using ::hadoop::hdfs::SetReplicationResponseProto;
  114. LOG_TRACE(kFileSystem,
  115. << "NameNodeOperations::SetReplication(" << FMT_THIS_ADDR << ", path=" << path <<
  116. ", replication=" << replication << ") called");
  117. if (path.empty()) {
  118. handler(Status::InvalidArgument("SetReplication: argument 'path' cannot be empty"));
  119. return;
  120. }
  121. Status replStatus = FileSystemImpl::CheckValidReplication(replication);
  122. if (!replStatus.ok()) {
  123. handler(replStatus);
  124. return;
  125. }
  126. SetReplicationRequestProto req;
  127. req.set_src(path);
  128. req.set_replication(replication);
  129. auto resp = std::make_shared<SetReplicationResponseProto>();
  130. namenode_.SetReplication(&req, resp, [resp, handler, path](const Status &stat) {
  131. if (stat.ok()) {
  132. // Checking resp
  133. if(resp -> has_result() && resp ->result() == 1) {
  134. handler(stat);
  135. } else {
  136. //NameNode does not specify why there is no result, in my testing it was happening when the path is not found
  137. std::string errormsg = "No such file or directory: " + path;
  138. Status statNew = Status::PathNotFound(errormsg.c_str());
  139. handler(statNew);
  140. }
  141. } else {
  142. handler(stat);
  143. }
  144. });
  145. }
  146. void NameNodeOperations::SetTimes(const std::string & path, uint64_t mtime, uint64_t atime,
  147. std::function<void(const Status &)> handler)
  148. {
  149. using ::hadoop::hdfs::SetTimesRequestProto;
  150. using ::hadoop::hdfs::SetTimesResponseProto;
  151. LOG_TRACE(kFileSystem,
  152. << "NameNodeOperations::SetTimes(" << FMT_THIS_ADDR << ", path=" << path <<
  153. ", mtime=" << mtime << ", atime=" << atime << ") called");
  154. if (path.empty()) {
  155. handler(Status::InvalidArgument("SetTimes: argument 'path' cannot be empty"));
  156. return;
  157. }
  158. SetTimesRequestProto req;
  159. req.set_src(path);
  160. req.set_mtime(mtime);
  161. req.set_atime(atime);
  162. auto resp = std::make_shared<SetTimesResponseProto>();
  163. namenode_.SetTimes(&req, resp, [resp, handler, path](const Status &stat) {
  164. handler(stat);
  165. });
  166. }
  167. void NameNodeOperations::GetFileInfo(const std::string & path,
  168. std::function<void(const Status &, const StatInfo &)> handler)
  169. {
  170. using ::hadoop::hdfs::GetFileInfoRequestProto;
  171. using ::hadoop::hdfs::GetFileInfoResponseProto;
  172. LOG_TRACE(kFileSystem, << "NameNodeOperations::GetFileInfo("
  173. << FMT_THIS_ADDR << ", path=" << path << ") called");
  174. if (path.empty()) {
  175. handler(Status::InvalidArgument("GetFileInfo: argument 'path' cannot be empty"), StatInfo());
  176. return;
  177. }
  178. GetFileInfoRequestProto req;
  179. req.set_src(path);
  180. auto resp = std::make_shared<GetFileInfoResponseProto>();
  181. namenode_.GetFileInfo(&req, resp, [resp, handler, path](const Status &stat) {
  182. if (stat.ok()) {
  183. // For non-existant files, the server will respond with an OK message but
  184. // no fs in the protobuf.
  185. if(resp -> has_fs()){
  186. struct StatInfo stat_info;
  187. stat_info.path = path;
  188. stat_info.full_path = path;
  189. HdfsFileStatusProtoToStatInfo(stat_info, resp->fs());
  190. handler(stat, stat_info);
  191. } else {
  192. std::string errormsg = "No such file or directory: " + path;
  193. Status statNew = Status::PathNotFound(errormsg.c_str());
  194. handler(statNew, StatInfo());
  195. }
  196. } else {
  197. handler(stat, StatInfo());
  198. }
  199. });
  200. }
  201. void NameNodeOperations::GetFsStats(
  202. std::function<void(const Status &, const FsInfo &)> handler) {
  203. using ::hadoop::hdfs::GetFsStatusRequestProto;
  204. using ::hadoop::hdfs::GetFsStatsResponseProto;
  205. LOG_TRACE(kFileSystem,
  206. << "NameNodeOperations::GetFsStats(" << FMT_THIS_ADDR << ") called");
  207. GetFsStatusRequestProto req;
  208. auto resp = std::make_shared<GetFsStatsResponseProto>();
  209. namenode_.GetFsStats(&req, resp, [resp, handler](const Status &stat) {
  210. if (stat.ok()) {
  211. struct FsInfo fs_info;
  212. GetFsStatsResponseProtoToFsInfo(fs_info, resp);
  213. handler(stat, fs_info);
  214. } else {
  215. handler(stat, FsInfo());
  216. }
  217. });
  218. }
  219. void NameNodeOperations::GetListing(
  220. const std::string & path,
  221. std::function<void(const Status &, const std::vector<StatInfo> &, bool)> handler,
  222. const std::string & start_after) {
  223. using ::hadoop::hdfs::GetListingRequestProto;
  224. using ::hadoop::hdfs::GetListingResponseProto;
  225. LOG_TRACE(
  226. kFileSystem,
  227. << "NameNodeOperations::GetListing(" << FMT_THIS_ADDR << ", path=" << path << ") called");
  228. if (path.empty()) {
  229. std::vector<StatInfo> empty;
  230. handler(Status::InvalidArgument("GetListing: argument 'path' cannot be empty"), empty, false);
  231. return;
  232. }
  233. GetListingRequestProto req;
  234. req.set_src(path);
  235. req.set_startafter(start_after.c_str());
  236. req.set_needlocation(false);
  237. auto resp = std::make_shared<GetListingResponseProto>();
  238. namenode_.GetListing(&req, resp, [resp, handler, path](const Status &stat) {
  239. std::vector<StatInfo> stat_infos;
  240. if (stat.ok()) {
  241. if(resp -> has_dirlist()){
  242. for (::hadoop::hdfs::HdfsFileStatusProto const& fs : resp->dirlist().partiallisting()) {
  243. StatInfo si;
  244. si.path = fs.path();
  245. si.full_path = path + fs.path() + "/";
  246. HdfsFileStatusProtoToStatInfo(si, fs);
  247. stat_infos.push_back(si);
  248. }
  249. handler(stat, stat_infos, resp->dirlist().remainingentries() > 0);
  250. } else {
  251. std::string errormsg = "No such file or directory: " + path;
  252. handler(Status::PathNotFound(errormsg.c_str()), stat_infos, false);
  253. }
  254. } else {
  255. handler(stat, stat_infos, false);
  256. }
  257. });
  258. }
  259. void NameNodeOperations::Mkdirs(const std::string & path, uint16_t permissions, bool createparent,
  260. std::function<void(const Status &)> handler)
  261. {
  262. using ::hadoop::hdfs::MkdirsRequestProto;
  263. using ::hadoop::hdfs::MkdirsResponseProto;
  264. LOG_TRACE(kFileSystem,
  265. << "NameNodeOperations::Mkdirs(" << FMT_THIS_ADDR << ", path=" << path <<
  266. ", permissions=" << permissions << ", createparent=" << createparent << ") called");
  267. if (path.empty()) {
  268. handler(Status::InvalidArgument("Mkdirs: argument 'path' cannot be empty"));
  269. return;
  270. }
  271. MkdirsRequestProto req;
  272. Status permStatus = FileSystemImpl::CheckValidPermissionMask(permissions);
  273. if (!permStatus.ok()) {
  274. handler(permStatus);
  275. return;
  276. }
  277. req.set_src(path);
  278. hadoop::hdfs::FsPermissionProto *perm = req.mutable_masked();
  279. perm->set_perm(permissions);
  280. req.set_createparent(createparent);
  281. auto resp = std::make_shared<MkdirsResponseProto>();
  282. namenode_.Mkdirs(&req, resp, [resp, handler, path](const Status &stat) {
  283. if (stat.ok()) {
  284. // Checking resp
  285. if(resp -> has_result() && resp ->result() == 1) {
  286. handler(stat);
  287. } else {
  288. //NameNode does not specify why there is no result, in my testing it was happening when the path is not found
  289. std::string errormsg = "No such file or directory: " + path;
  290. Status statNew = Status::PathNotFound(errormsg.c_str());
  291. handler(statNew);
  292. }
  293. } else {
  294. handler(stat);
  295. }
  296. });
  297. }
  298. void NameNodeOperations::Delete(const std::string & path, bool recursive, std::function<void(const Status &)> handler) {
  299. using ::hadoop::hdfs::DeleteRequestProto;
  300. using ::hadoop::hdfs::DeleteResponseProto;
  301. LOG_TRACE(kFileSystem,
  302. << "NameNodeOperations::Delete(" << FMT_THIS_ADDR << ", path=" << path << ", recursive=" << recursive << ") called");
  303. if (path.empty()) {
  304. handler(Status::InvalidArgument("Delete: argument 'path' cannot be empty"));
  305. return;
  306. }
  307. DeleteRequestProto req;
  308. req.set_src(path);
  309. req.set_recursive(recursive);
  310. auto resp = std::make_shared<DeleteResponseProto>();
  311. namenode_.Delete(&req, resp, [resp, handler, path](const Status &stat) {
  312. if (stat.ok()) {
  313. // Checking resp
  314. if(resp -> has_result() && resp ->result() == 1) {
  315. handler(stat);
  316. } else {
  317. //NameNode does not specify why there is no result, in my testing it was happening when the path is not found
  318. std::string errormsg = "No such file or directory: " + path;
  319. Status statNew = Status::PathNotFound(errormsg.c_str());
  320. handler(statNew);
  321. }
  322. } else {
  323. handler(stat);
  324. }
  325. });
  326. }
  327. void NameNodeOperations::Rename(const std::string & oldPath, const std::string & newPath, std::function<void(const Status &)> handler) {
  328. using ::hadoop::hdfs::RenameRequestProto;
  329. using ::hadoop::hdfs::RenameResponseProto;
  330. LOG_TRACE(kFileSystem,
  331. << "NameNodeOperations::Rename(" << FMT_THIS_ADDR << ", oldPath=" << oldPath << ", newPath=" << newPath << ") called");
  332. if (oldPath.empty()) {
  333. handler(Status::InvalidArgument("Rename: argument 'oldPath' cannot be empty"));
  334. return;
  335. }
  336. if (newPath.empty()) {
  337. handler(Status::InvalidArgument("Rename: argument 'newPath' cannot be empty"));
  338. return;
  339. }
  340. RenameRequestProto req;
  341. req.set_src(oldPath);
  342. req.set_dst(newPath);
  343. auto resp = std::make_shared<RenameResponseProto>();
  344. namenode_.Rename(&req, resp, [resp, handler](const Status &stat) {
  345. if (stat.ok()) {
  346. // Checking resp
  347. if(resp -> has_result() && resp ->result() == 1) {
  348. handler(stat);
  349. } else {
  350. //Since NameNode does not specify why the result is not success, we set the general error
  351. std::string errormsg = "oldPath and parent directory of newPath must exist. newPath must not exist.";
  352. Status statNew = Status::InvalidArgument(errormsg.c_str());
  353. handler(statNew);
  354. }
  355. } else {
  356. handler(stat);
  357. }
  358. });
  359. }
  360. void NameNodeOperations::SetPermission(const std::string & path,
  361. uint16_t permissions, std::function<void(const Status &)> handler) {
  362. using ::hadoop::hdfs::SetPermissionRequestProto;
  363. using ::hadoop::hdfs::SetPermissionResponseProto;
  364. LOG_TRACE(kFileSystem,
  365. << "NameNodeOperations::SetPermission(" << FMT_THIS_ADDR << ", path=" << path << ", permissions=" << permissions << ") called");
  366. if (path.empty()) {
  367. handler(Status::InvalidArgument("SetPermission: argument 'path' cannot be empty"));
  368. return;
  369. }
  370. Status permStatus = FileSystemImpl::CheckValidPermissionMask(permissions);
  371. if (!permStatus.ok()) {
  372. handler(permStatus);
  373. return;
  374. }
  375. SetPermissionRequestProto req;
  376. req.set_src(path);
  377. hadoop::hdfs::FsPermissionProto *perm = req.mutable_permission();
  378. perm->set_perm(permissions);
  379. auto resp = std::make_shared<SetPermissionResponseProto>();
  380. namenode_.SetPermission(&req, resp,
  381. [handler](const Status &stat) {
  382. handler(stat);
  383. });
  384. }
  385. void NameNodeOperations::SetOwner(const std::string & path,
  386. const std::string & username, const std::string & groupname, std::function<void(const Status &)> handler) {
  387. using ::hadoop::hdfs::SetOwnerRequestProto;
  388. using ::hadoop::hdfs::SetOwnerResponseProto;
  389. LOG_TRACE(kFileSystem,
  390. << "NameNodeOperations::SetOwner(" << FMT_THIS_ADDR << ", path=" << path << ", username=" << username << ", groupname=" << groupname << ") called");
  391. if (path.empty()) {
  392. handler(Status::InvalidArgument("SetOwner: argument 'path' cannot be empty"));
  393. return;
  394. }
  395. SetOwnerRequestProto req;
  396. req.set_src(path);
  397. if(!username.empty()) {
  398. req.set_username(username);
  399. }
  400. if(!groupname.empty()) {
  401. req.set_groupname(groupname);
  402. }
  403. auto resp = std::make_shared<SetOwnerResponseProto>();
  404. namenode_.SetOwner(&req, resp,
  405. [handler](const Status &stat) {
  406. handler(stat);
  407. });
  408. }
  409. void NameNodeOperations::CreateSnapshot(const std::string & path,
  410. const std::string & name, std::function<void(const Status &)> handler) {
  411. using ::hadoop::hdfs::CreateSnapshotRequestProto;
  412. using ::hadoop::hdfs::CreateSnapshotResponseProto;
  413. LOG_TRACE(kFileSystem,
  414. << "NameNodeOperations::CreateSnapshot(" << FMT_THIS_ADDR << ", path=" << path << ", name=" << name << ") called");
  415. if (path.empty()) {
  416. handler(Status::InvalidArgument("CreateSnapshot: argument 'path' cannot be empty"));
  417. return;
  418. }
  419. CreateSnapshotRequestProto req;
  420. req.set_snapshotroot(path);
  421. if (!name.empty()) {
  422. req.set_snapshotname(name);
  423. }
  424. auto resp = std::make_shared<CreateSnapshotResponseProto>();
  425. namenode_.CreateSnapshot(&req, resp,
  426. [handler](const Status &stat) {
  427. handler(stat);
  428. });
  429. }
  430. void NameNodeOperations::DeleteSnapshot(const std::string & path,
  431. const std::string & name, std::function<void(const Status &)> handler) {
  432. using ::hadoop::hdfs::DeleteSnapshotRequestProto;
  433. using ::hadoop::hdfs::DeleteSnapshotResponseProto;
  434. LOG_TRACE(kFileSystem,
  435. << "NameNodeOperations::DeleteSnapshot(" << FMT_THIS_ADDR << ", path=" << path << ", name=" << name << ") called");
  436. if (path.empty()) {
  437. handler(Status::InvalidArgument("DeleteSnapshot: argument 'path' cannot be empty"));
  438. return;
  439. }
  440. if (name.empty()) {
  441. handler(Status::InvalidArgument("DeleteSnapshot: argument 'name' cannot be empty"));
  442. return;
  443. }
  444. DeleteSnapshotRequestProto req;
  445. req.set_snapshotroot(path);
  446. req.set_snapshotname(name);
  447. auto resp = std::make_shared<DeleteSnapshotResponseProto>();
  448. namenode_.DeleteSnapshot(&req, resp,
  449. [handler](const Status &stat) {
  450. handler(stat);
  451. });
  452. }
  453. void NameNodeOperations::AllowSnapshot(const std::string & path, std::function<void(const Status &)> handler) {
  454. using ::hadoop::hdfs::AllowSnapshotRequestProto;
  455. using ::hadoop::hdfs::AllowSnapshotResponseProto;
  456. LOG_TRACE(kFileSystem,
  457. << "NameNodeOperations::AllowSnapshot(" << FMT_THIS_ADDR << ", path=" << path << ") called");
  458. if (path.empty()) {
  459. handler(Status::InvalidArgument("AllowSnapshot: argument 'path' cannot be empty"));
  460. return;
  461. }
  462. AllowSnapshotRequestProto req;
  463. req.set_snapshotroot(path);
  464. auto resp = std::make_shared<AllowSnapshotResponseProto>();
  465. namenode_.AllowSnapshot(&req, resp,
  466. [handler](const Status &stat) {
  467. handler(stat);
  468. });
  469. }
  470. void NameNodeOperations::DisallowSnapshot(const std::string & path, std::function<void(const Status &)> handler) {
  471. using ::hadoop::hdfs::DisallowSnapshotRequestProto;
  472. using ::hadoop::hdfs::DisallowSnapshotResponseProto;
  473. LOG_TRACE(kFileSystem,
  474. << "NameNodeOperations::DisallowSnapshot(" << FMT_THIS_ADDR << ", path=" << path << ") called");
  475. if (path.empty()) {
  476. handler(Status::InvalidArgument("DisallowSnapshot: argument 'path' cannot be empty"));
  477. return;
  478. }
  479. DisallowSnapshotRequestProto req;
  480. req.set_snapshotroot(path);
  481. auto resp = std::make_shared<DisallowSnapshotResponseProto>();
  482. namenode_.DisallowSnapshot(&req, resp,
  483. [handler](const Status &stat) {
  484. handler(stat);
  485. });
  486. }
  487. void NameNodeOperations::SetFsEventCallback(fs_event_callback callback) {
  488. engine_.SetFsEventCallback(callback);
  489. }
  490. void NameNodeOperations::HdfsFileStatusProtoToStatInfo(
  491. hdfs::StatInfo & stat_info,
  492. const ::hadoop::hdfs::HdfsFileStatusProto & fs) {
  493. stat_info.file_type = fs.filetype();
  494. stat_info.length = fs.length();
  495. stat_info.permissions = fs.permission().perm();
  496. stat_info.owner = fs.owner();
  497. stat_info.group = fs.group();
  498. stat_info.modification_time = fs.modification_time();
  499. stat_info.access_time = fs.access_time();
  500. stat_info.symlink = fs.symlink();
  501. stat_info.block_replication = fs.block_replication();
  502. stat_info.blocksize = fs.blocksize();
  503. stat_info.fileid = fs.fileid();
  504. stat_info.children_num = fs.childrennum();
  505. }
  506. void NameNodeOperations::GetFsStatsResponseProtoToFsInfo(
  507. hdfs::FsInfo & fs_info,
  508. const std::shared_ptr<::hadoop::hdfs::GetFsStatsResponseProto> & fs) {
  509. fs_info.capacity = fs->capacity();
  510. fs_info.used = fs->used();
  511. fs_info.remaining = fs->remaining();
  512. fs_info.under_replicated = fs->under_replicated();
  513. fs_info.corrupt_blocks = fs->corrupt_blocks();
  514. fs_info.missing_blocks = fs->missing_blocks();
  515. fs_info.missing_repl_one_blocks = fs->missing_repl_one_blocks();
  516. if(fs->has_blocks_in_future()){
  517. fs_info.blocks_in_future = fs->blocks_in_future();
  518. }
  519. }
  520. }