datanodeconnection.h 2.7 KB

1234567891011121314151617181920212223242526272829303132333435363738394041424344454647484950515253545556575859606162636465666768697071727374757677
  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. #ifndef LIBHDFSPP_LIB_CONNECTION_DATANODECONNECTION_H_
  19. #define LIBHDFSPP_LIB_CONNECTION_DATANODECONNECTION_H_
  20. #include "common/hdfs_public_api.h"
  21. #include "common/async_stream.h"
  22. #include "ClientNamenodeProtocol.pb.h"
  23. #include "common/libhdfs_events_impl.h"
  24. #include "asio.hpp"
  25. namespace hdfs {
  26. class DataNodeConnection : public AsyncStream {
  27. public:
  28. std::string uuid_;
  29. std::unique_ptr<hadoop::common::TokenProto> token_;
  30. virtual ~DataNodeConnection();
  31. virtual void Connect(std::function<void(Status status, std::shared_ptr<DataNodeConnection> dn)> handler) = 0;
  32. virtual void Cancel() = 0;
  33. };
  34. class DataNodeConnectionImpl : public DataNodeConnection, public std::enable_shared_from_this<DataNodeConnectionImpl>{
  35. public:
  36. std::unique_ptr<asio::ip::tcp::socket> conn_;
  37. std::array<asio::ip::tcp::endpoint, 1> endpoints_;
  38. std::string uuid_;
  39. LibhdfsEvents *event_handlers_;
  40. virtual ~DataNodeConnectionImpl();
  41. DataNodeConnectionImpl(asio::io_service * io_service, const ::hadoop::hdfs::DatanodeInfoProto &dn_proto,
  42. const hadoop::common::TokenProto *token,
  43. LibhdfsEvents *event_handlers);
  44. void Connect(std::function<void(Status status, std::shared_ptr<DataNodeConnection> dn)> handler) override;
  45. void Cancel() override;
  46. void async_read_some(const MutableBuffers &buf,
  47. std::function<void (const asio::error_code & error,
  48. std::size_t bytes_transferred) > handler) override {
  49. event_handlers_->call("DN_read_req", "", "", buf.end() - buf.begin());
  50. conn_->async_read_some(buf, handler);
  51. };
  52. void async_write_some(const ConstBuffers &buf,
  53. std::function<void (const asio::error_code & error,
  54. std::size_t bytes_transferred) > handler) override {
  55. event_handlers_->call("DN_write_req", "", "", buf.end() - buf.begin());
  56. conn_->async_write_some(buf, handler);
  57. }
  58. };
  59. }
  60. #endif