asio.h 3.4 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116
  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 LIB_COMMON_COROUTINES_ASIO_H_
  19. #define LIB_COMMON_COROUTINES_ASIO_H_
  20. #include "continuation.h"
  21. #include "common/util.h"
  22. #include "libhdfs++/status.h"
  23. #include <asio/connect.hpp>
  24. #include <asio/read.hpp>
  25. #include <asio/write.hpp>
  26. #include <asio/ip/tcp.hpp>
  27. namespace hdfs {
  28. namespace continuation {
  29. template<class Stream, class MutableBufferSequence>
  30. class ReadContinuation : public Continuation {
  31. public:
  32. ReadContinuation(Stream *stream, const MutableBufferSequence &buffer)
  33. : stream_(stream)
  34. , buffer_(buffer)
  35. {}
  36. virtual void Run(const Next& next) override {
  37. auto handler = [next](const asio::error_code &ec, size_t)
  38. { next(ToStatus(ec)); };
  39. asio::async_read(*stream_, buffer_, handler);
  40. }
  41. private:
  42. Stream *stream_;
  43. MutableBufferSequence buffer_;
  44. };
  45. template<class Stream, class ConstBufferSequence>
  46. class WriteContinuation : public Continuation {
  47. public:
  48. WriteContinuation(Stream *stream, const ConstBufferSequence& buffer)
  49. : stream_(stream)
  50. , buffer_(buffer)
  51. {}
  52. virtual void Run(const Next& next) override {
  53. auto handler = [next](const asio::error_code &ec, size_t)
  54. { next(ToStatus(ec)); };
  55. asio::async_write(*stream_, buffer_, handler);
  56. }
  57. private:
  58. Stream *stream_;
  59. ConstBufferSequence buffer_;
  60. };
  61. template<class Socket, class Iterator>
  62. class ConnectContinuation : public Continuation {
  63. public:
  64. ConnectContinuation(Socket *socket, Iterator begin, Iterator end,
  65. Iterator *connected_endpoint)
  66. : socket_(socket)
  67. , begin_(begin)
  68. , end_(end)
  69. , connected_endpoint_(connected_endpoint)
  70. {}
  71. virtual void Run(const Next& next) override {
  72. auto handler = [this,next](const asio::error_code &ec, Iterator it) {
  73. if (connected_endpoint_) {
  74. *connected_endpoint_ = it;
  75. }
  76. next(ToStatus(ec));
  77. };
  78. asio::async_connect(*socket_, begin_, end_, handler);
  79. }
  80. private:
  81. Socket *socket_;
  82. Iterator begin_;
  83. Iterator end_;
  84. Iterator *connected_endpoint_;
  85. };
  86. template<class Stream, class ConstBufferSequence>
  87. static inline Continuation* Write(Stream *stream, const ConstBufferSequence &buffer) {
  88. return new WriteContinuation<Stream, ConstBufferSequence>(stream, buffer);
  89. }
  90. template<class Stream, class MutableBufferSequence>
  91. static inline Continuation* Read(Stream *stream, const MutableBufferSequence &buffer) {
  92. return new ReadContinuation<Stream, MutableBufferSequence>(stream, buffer);
  93. }
  94. template<class Socket, class Iterator>
  95. static inline Continuation* Connect(Socket *socket, Iterator begin, Iterator end) {
  96. return new ConnectContinuation<Socket, Iterator>(socket, begin, end, nullptr);
  97. }
  98. }
  99. }
  100. #endif