}
-seastar::future<> AdminSocket::handle_command(crimson::net::Connection* conn,
+seastar::future<> AdminSocket::handle_command(crimson::net::ConnectionRef conn,
boost::intrusive_ptr<MCommand> m)
{
return execute_command(m->cmd, std::move(m->get_data())).then(
* \param conn connection over which the incoming command message is received
* \param m message carrying the command vector and optional input buffer
*/
- seastar::future<> handle_command(crimson::net::Connection* conn,
+ seastar::future<> handle_command(crimson::net::ConnectionRef conn,
boost::intrusive_ptr<MCommand> m);
private:
}
std::tuple<bool, seastar::future<>>
-Client::ms_dispatch(crimson::net::Connection* conn, MessageRef m)
+Client::ms_dispatch(crimson::net::ConnectionRef conn, MessageRef m)
{
bool dispatched = true;
gate.dispatch_in_background(__func__, *this, [this, conn, &m, &dispatched] {
});
}
-seastar::future<> Client::handle_mgr_map(crimson::net::Connection*,
+seastar::future<> Client::handle_mgr_map(crimson::net::ConnectionRef,
Ref<MMgrMap> m)
{
mgrmap = m->get_map();
}
}
-seastar::future<> Client::handle_mgr_conf(crimson::net::Connection* conn,
+seastar::future<> Client::handle_mgr_conf(crimson::net::ConnectionRef,
Ref<MMgrConfigure> m)
{
logger().info("{} {}", __func__, *m);
private:
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* conn, Ref<Message> m) override;
+ crimson::net::ConnectionRef conn, Ref<Message> m) override;
void ms_handle_reset(crimson::net::ConnectionRef conn, bool is_replace) final;
void ms_handle_connect(crimson::net::ConnectionRef conn) final;
- seastar::future<> handle_mgr_map(crimson::net::Connection* conn,
+ seastar::future<> handle_mgr_map(crimson::net::ConnectionRef conn,
Ref<MMgrMap> m);
- seastar::future<> handle_mgr_conf(crimson::net::Connection* conn,
+ seastar::future<> handle_mgr_conf(crimson::net::ConnectionRef conn,
Ref<MMgrConfigure> m);
seastar::future<> reconnect();
}
std::tuple<bool, seastar::future<>>
-Client::ms_dispatch(crimson::net::Connection* conn, MessageRef m)
+Client::ms_dispatch(crimson::net::ConnectionRef conn, MessageRef m)
{
bool dispatched = true;
gate.dispatch_in_background(__func__, *this, [this, conn, &m, &dispatched] {
}
}
-seastar::future<> Client::handle_monmap(crimson::net::Connection* conn,
+seastar::future<> Client::handle_monmap(crimson::net::ConnectionRef conn,
Ref<MMonMap> m)
{
monmap.decode(m->monmapbl);
}
}
-seastar::future<> Client::handle_auth_reply(crimson::net::Connection* conn,
- Ref<MAuthReply> m)
+seastar::future<> Client::handle_auth_reply(crimson::net::ConnectionRef conn,
+ Ref<MAuthReply> m)
{
logger().info(
"handle_auth_reply mon {} => {} returns {}: {}",
private:
void tick();
- std::tuple<bool, seastar::future<>> ms_dispatch(crimson::net::Connection* conn,
+ std::tuple<bool, seastar::future<>> ms_dispatch(crimson::net::ConnectionRef conn,
MessageRef m) override;
void ms_handle_reset(crimson::net::ConnectionRef conn, bool is_replace) override;
- seastar::future<> handle_monmap(crimson::net::Connection* conn,
+ seastar::future<> handle_monmap(crimson::net::ConnectionRef conn,
Ref<MMonMap> m);
- seastar::future<> handle_auth_reply(crimson::net::Connection* conn,
+ seastar::future<> handle_auth_reply(crimson::net::ConnectionRef conn,
Ref<MAuthReply> m);
seastar::future<> handle_subscribe_ack(Ref<MMonSubscribeAck> m);
seastar::future<> handle_get_version_reply(Ref<MMonGetVersionReply> m);
auto get_last_keepalive() const { return last_keepalive; }
auto get_last_keepalive_ack() const { return last_keepalive_ack; }
- seastar::shared_ptr<Connection> get_shared() {
- return shared_from_this();
- }
-
struct user_private_t {
virtual ~user_private_t() = default;
};
// to prevent other dispatchers from processing it, and returns a future
// to throttle the connection if it's too busy. Else, it returns false and
// the second future is ignored.
- virtual std::tuple<bool, seastar::future<>> ms_dispatch(Connection*, MessageRef) = 0;
+ virtual std::tuple<bool, seastar::future<>> ms_dispatch(ConnectionRef, MessageRef) = 0;
virtual void ms_handle_accept(ConnectionRef conn) {}
}).then([this] (bufferlist bl) {
auto p = bl.cbegin();
::decode(m.footer, p);
- auto pconn = seastar::static_pointer_cast<SocketConnection>(
+ auto conn_ref = seastar::static_pointer_cast<SocketConnection>(
conn.shared_from_this());
auto msg = ::decode_message(nullptr, 0, m.header, m.footer,
- m.front, m.middle, m.data, std::move(pconn));
+ m.front, m.middle, m.data, conn_ref);
if (unlikely(!msg)) {
logger().warn("{} decode message failed", conn);
throw std::system_error{make_error_code(error::corrupted_message)};
logger().debug("{} <== #{} === {} ({})",
conn, msg_ref->get_seq(), *msg_ref, msg_ref->get_type());
// throttle the reading process by the returned future
- return dispatchers.ms_dispatch(&conn, std::move(msg_ref));
+ return dispatchers.ms_dispatch(conn_ref, std::move(msg_ref));
});
}
ceph_msg_footer footer{init_le32(0), init_le32(0),
init_le32(0), init_le64(0), current_header.flags};
- auto pconn = seastar::static_pointer_cast<SocketConnection>(
+ auto conn_ref = seastar::static_pointer_cast<SocketConnection>(
conn.shared_from_this());
Message *message = decode_message(nullptr, 0, header, footer,
- msg_frame.front(), msg_frame.middle(), msg_frame.data(),
- std::move(pconn));
+ msg_frame.front(), msg_frame.middle(), msg_frame.data(), conn_ref);
if (!message) {
logger().warn("{} decode message failed", conn);
abort_in_fault();
// TODO: change MessageRef with seastar::shared_ptr
auto msg_ref = MessageRef{message, false};
// throttle the reading process by the returned future
- return dispatchers.ms_dispatch(&conn, std::move(msg_ref));
+ return dispatchers.ms_dispatch(conn_ref, std::move(msg_ref));
});
}
namespace crimson::net {
seastar::future<>
-ChainedDispatchers::ms_dispatch(crimson::net::Connection* conn,
+ChainedDispatchers::ms_dispatch(crimson::net::ConnectionRef conn,
MessageRef m) {
try {
for (auto& dispatcher : dispatchers) {
bool empty() const {
return dispatchers.empty();
}
- seastar::future<> ms_dispatch(crimson::net::Connection* conn, MessageRef m);
+ seastar::future<> ms_dispatch(crimson::net::ConnectionRef, MessageRef);
void ms_handle_accept(crimson::net::ConnectionRef conn);
void ms_handle_connect(crimson::net::ConnectionRef conn);
void ms_handle_reset(crimson::net::ConnectionRef conn, bool is_replace);
}
std::tuple<bool, seastar::future<>>
-Heartbeat::ms_dispatch(crimson::net::Connection* conn, MessageRef m)
+Heartbeat::ms_dispatch(crimson::net::ConnectionRef conn, MessageRef m)
{
bool dispatched = true;
gate.dispatch_in_background(__func__, *this, [this, conn, &m, &dispatched] {
}
}
-seastar::future<> Heartbeat::handle_osd_ping(crimson::net::Connection* conn,
+seastar::future<> Heartbeat::handle_osd_ping(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m)
{
switch (m->op) {
}
}
-seastar::future<> Heartbeat::handle_ping(crimson::net::Connection* conn,
+seastar::future<> Heartbeat::handle_ping(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m)
{
auto min_message = static_cast<uint32_t>(
return conn->send(reply);
}
-seastar::future<> Heartbeat::handle_reply(crimson::net::Connection* conn,
+seastar::future<> Heartbeat::handle_reply(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m)
{
const osd_id_t from = m->get_source().num();
}
}
-bool Heartbeat::Connection::matches(crimson::net::Connection* _conn) const
+bool Heartbeat::Connection::matches(crimson::net::ConnectionRef _conn) const
{
- return (conn && conn.get() == _conn);
+ return (conn && conn == _conn);
}
void Heartbeat::Connection::accepted(crimson::net::ConnectionRef accepted_conn)
}
seastar::future<> Heartbeat::Peer::handle_reply(
- crimson::net::Connection* conn, Ref<MOSDPing> m)
+ crimson::net::ConnectionRef conn, Ref<MOSDPing> m)
{
if (!session.is_started()) {
// we haven't sent any ping yet
// Dispatcher methods
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* conn, MessageRef m) override;
+ crimson::net::ConnectionRef conn, MessageRef m) override;
void ms_handle_reset(crimson::net::ConnectionRef conn, bool is_replace) override;
void ms_handle_connect(crimson::net::ConnectionRef conn) override;
void ms_handle_accept(crimson::net::ConnectionRef conn) override;
void print(std::ostream&) const;
private:
- seastar::future<> handle_osd_ping(crimson::net::Connection* conn,
+ seastar::future<> handle_osd_ping(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m);
- seastar::future<> handle_ping(crimson::net::Connection* conn,
+ seastar::future<> handle_ping(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m);
- seastar::future<> handle_reply(crimson::net::Connection* conn,
+ seastar::future<> handle_reply(crimson::net::ConnectionRef conn,
Ref<MOSDPing> m);
seastar::future<> handle_you_died();
~Connection();
- bool matches(crimson::net::Connection* _conn) const;
- bool matches(crimson::net::ConnectionRef conn) const {
- return matches(conn.get());
- }
+ bool matches(crimson::net::ConnectionRef _conn) const;
void connected() {
set_connected();
}
}
void send_heartbeat(
clock::time_point, ceph::signedspan, std::vector<seastar::future<>>&);
- seastar::future<> handle_reply(crimson::net::Connection*, Ref<MOSDPing>);
+ seastar::future<> handle_reply(crimson::net::ConnectionRef, Ref<MOSDPing>);
void handle_reset(crimson::net::ConnectionRef conn, bool is_replace) {
for_each_conn([&] (auto& _conn) {
if (_conn.matches(conn)) {
});
}
-seastar::future<> OSD::handle_command(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_command(crimson::net::ConnectionRef conn,
Ref<MCommand> m)
{
return asok->handle_command(conn, std::move(m));
}
std::tuple<bool, seastar::future<>>
-OSD::ms_dispatch(crimson::net::Connection* conn, MessageRef m)
+OSD::ms_dispatch(crimson::net::ConnectionRef conn, MessageRef m)
{
if (state.is_stopping()) {
return {false, seastar::now()};
case MSG_OSD_PG_CREATE2:
shard_services.start_operation<CompoundPeeringRequest>(
*this,
- conn->get_shared(),
+ conn,
m);
return seastar::now();
case MSG_COMMAND:
});
}
-seastar::future<> OSD::handle_osd_map(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_osd_map(crimson::net::ConnectionRef conn,
Ref<MOSDMap> m)
{
logger().info("handle_osd_map {}", *m);
});
}
-seastar::future<> OSD::handle_osd_op(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_osd_op(crimson::net::ConnectionRef conn,
Ref<MOSDOp> m)
{
(void) shard_services.start_operation<ClientRequest>(
*this,
- conn->get_shared(),
+ conn,
std::move(m));
return seastar::now();
}
-seastar::future<> OSD::send_incremental_map(crimson::net::Connection* conn,
+seastar::future<> OSD::send_incremental_map(crimson::net::ConnectionRef conn,
epoch_t first)
{
if (first >= superblock.oldest_map) {
}
}
-seastar::future<> OSD::handle_rep_op(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_rep_op(crimson::net::ConnectionRef conn,
Ref<MOSDRepOp> m)
{
m->finish_decode();
(void) shard_services.start_operation<RepRequest>(
*this,
- conn->get_shared(),
+ std::move(conn),
std::move(m));
return seastar::now();
}
-seastar::future<> OSD::handle_rep_op_reply(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_rep_op_reply(crimson::net::ConnectionRef conn,
Ref<MOSDRepOpReply> m)
{
const auto& pgs = pg_map.get_pgs();
return seastar::now();
}
-seastar::future<> OSD::handle_scrub(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_scrub(crimson::net::ConnectionRef conn,
Ref<MOSDScrub2> m)
{
if (m->fsid != superblock.cluster_fsid) {
return seastar::now();
}
return seastar::parallel_for_each(std::move(m->scrub_pgs),
- [m, conn=conn->get_shared(), this](spg_t pgid) {
+ [m, conn, this](spg_t pgid) {
pg_shard_t from_shard{static_cast<int>(m->get_source().num()),
pgid.shard};
PeeringState::RequestScrub scrub_request{m->deep, m->repair};
});
}
-seastar::future<> OSD::handle_mark_me_down(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_mark_me_down(crimson::net::ConnectionRef conn,
Ref<MOSDMarkMeDown> m)
{
if (state.is_prestop()) {
return seastar::now();
}
-seastar::future<> OSD::handle_recovery_subreq(crimson::net::Connection* conn,
+seastar::future<> OSD::handle_recovery_subreq(crimson::net::ConnectionRef conn,
Ref<MOSDFastDispatchOp> m)
{
(void) shard_services.start_operation<RecoverySubRequest>(
*this,
- conn->get_shared(),
+ conn,
std::move(m));
return seastar::now();
}
}
seastar::future<> OSD::handle_peering_op(
- crimson::net::Connection* conn,
+ crimson::net::ConnectionRef conn,
Ref<MOSDPeeringOp> m)
{
const int from = m->get_source().num();
std::unique_ptr<PGPeeringEvent> evt(m->get_event());
(void) shard_services.start_operation<RemotePeeringEvent>(
*this,
- conn->get_shared(),
+ conn,
shard_services,
pg_shard_t{from, m->get_spg().shard},
m->get_spg(),
OSDSuperblock superblock;
// Dispatcher methods
- std::tuple<bool, seastar::future<>> ms_dispatch(crimson::net::Connection*, MessageRef) final;
+ std::tuple<bool, seastar::future<>> ms_dispatch(crimson::net::ConnectionRef, MessageRef) final;
void ms_handle_reset(crimson::net::ConnectionRef conn, bool is_replace) final;
void ms_handle_remote_reset(crimson::net::ConnectionRef conn) final;
void dump_pg_state_history(Formatter*) const;
void print(std::ostream&) const;
- seastar::future<> send_incremental_map(crimson::net::Connection* conn,
+ seastar::future<> send_incremental_map(crimson::net::ConnectionRef conn,
epoch_t first);
/// @return the seq id of the pg stats being sent
seastar::future<Ref<PG>> handle_pg_create_info(
std::unique_ptr<PGCreateInfo> info);
- seastar::future<> handle_osd_map(crimson::net::Connection* conn,
+ seastar::future<> handle_osd_map(crimson::net::ConnectionRef conn,
Ref<MOSDMap> m);
- seastar::future<> handle_osd_op(crimson::net::Connection* conn,
+ seastar::future<> handle_osd_op(crimson::net::ConnectionRef conn,
Ref<MOSDOp> m);
- seastar::future<> handle_rep_op(crimson::net::Connection* conn,
+ seastar::future<> handle_rep_op(crimson::net::ConnectionRef conn,
Ref<MOSDRepOp> m);
- seastar::future<> handle_rep_op_reply(crimson::net::Connection* conn,
+ seastar::future<> handle_rep_op_reply(crimson::net::ConnectionRef conn,
Ref<MOSDRepOpReply> m);
- seastar::future<> handle_peering_op(crimson::net::Connection* conn,
+ seastar::future<> handle_peering_op(crimson::net::ConnectionRef conn,
Ref<MOSDPeeringOp> m);
- seastar::future<> handle_recovery_subreq(crimson::net::Connection* conn,
+ seastar::future<> handle_recovery_subreq(crimson::net::ConnectionRef conn,
Ref<MOSDFastDispatchOp> m);
- seastar::future<> handle_scrub(crimson::net::Connection* conn,
+ seastar::future<> handle_scrub(crimson::net::ConnectionRef conn,
Ref<MOSDScrub2> m);
- seastar::future<> handle_mark_me_down(crimson::net::Connection* conn,
+ seastar::future<> handle_mark_me_down(crimson::net::ConnectionRef conn,
Ref<MOSDMarkMeDown> m);
seastar::future<> committed_osd_maps(version_t first,
void check_osdmap_features();
- seastar::future<> handle_command(crimson::net::Connection* conn,
+ seastar::future<> handle_command(crimson::net::ConnectionRef conn,
Ref<MCommand> m);
seastar::future<> start_asok_admin();
}).then([this, opref](Ref<PG> pgref) {
PG &pg = *pgref;
if (pg.can_discard_op(*m)) {
- return osd.send_incremental_map(conn.get(), m->get_map_epoch());
+ return osd.send_incremental_map(conn, m->get_map_epoch());
}
return with_blocking_future(
handle.enter(pp(pg).await_map)
});
}
-void PG::handle_rep_op_reply(crimson::net::Connection* conn,
+void PG::handle_rep_op_reply(crimson::net::ConnectionRef conn,
const MOSDRepOpReply& m)
{
if (!can_discard_replica_op(m)) {
with_obc_func_t&& f);
seastar::future<> handle_rep_op(Ref<MOSDRepOp> m);
- void handle_rep_op_reply(crimson::net::Connection* conn,
+ void handle_rep_op_reply(crimson::net::ConnectionRef conn,
const MOSDRepOpReply& m);
void print(std::ostream& os) const;
crimson::auth::DummyAuthClientServer dummy_auth;
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef c, MessageRef m) override {
if (verbose) {
logger().info("server got {}", *m);
}
unsigned rounds;
std::bernoulli_distribution keepalive_dist;
crimson::net::MessengerRef msgr;
- std::map<crimson::net::Connection*, seastar::promise<>> pending_conns;
- std::map<crimson::net::Connection*, PingSessionRef> sessions;
+ std::map<crimson::net::ConnectionRef, seastar::promise<>> pending_conns;
+ std::map<crimson::net::ConnectionRef, PingSessionRef> sessions;
crimson::auth::DummyAuthClientServer dummy_auth;
Client(unsigned rounds, double keepalive_ratio)
: rounds(rounds),
keepalive_dist(std::bernoulli_distribution{keepalive_ratio}) {}
- PingSessionRef find_session(crimson::net::Connection* c) {
+ PingSessionRef find_session(crimson::net::ConnectionRef c) {
auto found = sessions.find(c);
if (found == sessions.end()) {
ceph_assert(false);
void ms_handle_connect(crimson::net::ConnectionRef conn) override {
auto session = seastar::make_shared<PingSession>();
- auto [i, added] = sessions.emplace(conn.get(), session);
+ auto [i, added] = sessions.emplace(conn, session);
std::ignore = i;
ceph_assert(added);
session->connected_time = mono_clock::now();
}
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef c, MessageRef m) override {
auto session = find_session(c);
++(session->count);
if (verbose) {
mono_time start_time = mono_clock::now();
auto conn = msgr->connect(peer_addr, entity_name_t::TYPE_OSD);
return seastar::futurize_invoke([this, conn] {
- return do_dispatch_pingpong(conn.get());
+ return do_dispatch_pingpong(conn);
}).then([this, conn, start_time] {
- auto session = find_session(conn.get());
+ auto session = find_session(conn);
std::chrono::duration<double> dur_handshake = session->connected_time - start_time;
std::chrono::duration<double> dur_pingpong = session->finish_time - session->connected_time;
logger().info("{}: handshake {}, pingpong {}",
}
private:
- seastar::future<> do_dispatch_pingpong(crimson::net::Connection* conn) {
+ seastar::future<> do_dispatch_pingpong(crimson::net::ConnectionRef conn) {
auto [i, added] = pending_conns.emplace(conn, seastar::promise<>());
std::ignore = i;
ceph_assert(added);
crimson::auth::DummyAuthClientServer dummy_auth;
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef, MessageRef m) override {
switch (++count) {
case 1:
// block on the first request until we reenter with the second
crimson::auth::DummyAuthClientServer dummy_auth;
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef, MessageRef m) override {
return {true, seastar::now()};
}
crimson::auth::DummyAuthClientServer dummy_auth;
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef c, MessageRef m) override {
std::ignore = c->send(make_message<MPing>());
return {true, seastar::now()};
}
seastar::promise<> stopped_send_promise;
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef, MessageRef m) override {
return {true, seastar::now()};
}
unsigned pending_peer_receive = 0;
unsigned pending_receive = 0;
- std::tuple<bool, seastar::future<>> ms_dispatch(Connection* c, MessageRef m) override {
- auto result = interceptor.find_result(c->shared_from_this());
+ std::tuple<bool, seastar::future<>> ms_dispatch(ConnectionRef c, MessageRef m) override {
+ auto result = interceptor.find_result(c);
if (result == nullptr) {
logger().error("Untracked ms dispatched connection: {}", *c);
ceph_abort();
}
- if (tracked_conn != c->shared_from_this()) {
+ if (tracked_conn != c) {
logger().error("[{}] {} got op, but doesn't match tracked_conn [{}] {}",
result->index, *c, tracked_index, *tracked_conn);
ceph_abort();
std::unique_ptr<FailoverSuite> test_suite;
- std::tuple<bool, seastar::future<>> ms_dispatch(Connection* c, MessageRef m) override {
+ std::tuple<bool, seastar::future<>> ms_dispatch(ConnectionRef c, MessageRef m) override {
switch (m->get_type()) {
case CEPH_MSG_PING:
ceph_assert(recv_pong);
ConnectionRef tracked_conn;
unsigned pending_send = 0;
- std::tuple<bool, seastar::future<>> ms_dispatch(Connection* c, MessageRef m) override {
+ std::tuple<bool, seastar::future<>> ms_dispatch(ConnectionRef c, MessageRef m) override {
logger().info("[TestPeer] got op from Test");
ceph_assert(m->get_type() == CEPH_MSG_OSD_OP);
- ceph_assert(tracked_conn == c->shared_from_this());
+ ceph_assert(tracked_conn == c);
std::ignore = op_callback();
return {true, seastar::now()};
}
const entity_addr_t test_peer_addr;
std::unique_ptr<FailoverSuitePeer> test_suite;
- std::tuple<bool, seastar::future<>> ms_dispatch(Connection* c, MessageRef m) override {
- ceph_assert(cmd_conn == c->shared_from_this());
+ std::tuple<bool, seastar::future<>> ms_dispatch(ConnectionRef c, MessageRef m) override {
+ ceph_assert(cmd_conn == c);
switch (m->get_type()) {
case CEPH_MSG_PING:
std::ignore = c->send(make_message<MPing>());
}
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef c, MessageRef m) override {
ceph_assert(m->get_type() == CEPH_MSG_OSD_OP);
// server replies with MOSDOp to generate server-side write workload
conn_stats.connected_time = mono_clock::now();
}
std::tuple<bool, seastar::future<>> ms_dispatch(
- crimson::net::Connection* c, MessageRef m) override {
+ crimson::net::ConnectionRef, MessageRef m) override {
// server replies with MOSDOp to generate server-side write workload
ceph_assert(m->get_type() == CEPH_MSG_OSD_OP);