Remove all targets from log statements. (#195)

The default uses crate + module path anyway, so `target` has been
redundant, causes more work when renaming crates and makes log
lines longer.
This commit is contained in:
Toralf Wittner 2018-05-17 15:14:13 +02:00 committed by GitHub
parent cb800624f5
commit 86a21fc43e
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
22 changed files with 158 additions and 217 deletions

View File

@ -172,11 +172,11 @@ where
let inner = self.inner;
let future = substream.and_then(move |outbound| {
if let Some(o) = outbound {
debug!(target: "libp2p-core", "Using existing multiplexed connection to {}", addr);
debug!("Using existing multiplexed connection to {}", addr);
return Either::A(future::ok(o));
}
// The previous stream muxer did not yield a new substream => start new dial
debug!(target: "libp2p-core", "No existing connection to {}; dialing", addr);
debug!("No existing connection to {}; dialing", addr);
match inner.dial(addr.clone()) {
Ok(dial) => {
let future = dial.and_then(move |(muxer, addr)| {
@ -193,7 +193,7 @@ where
));
Ok((s, addr))
} else {
error!(target: "libp2p-core", "failed to dial to {}", addr);
error!("failed to dial to {}", addr);
shared.lock().active_connections.remove(&addr);
Err(io::Error::new(io::ErrorKind::Other, "dial failed"))
}
@ -273,13 +273,13 @@ where
}
Ok(Async::NotReady) => {}
Ok(Async::Ready(None)) => {
debug!(target: "libp2p-core", "listener has been closed");
debug!("listener has been closed");
if self.connections.is_empty() && self.current_upgrades.is_empty() {
return Ok(Async::Ready(None));
}
}
Err(err) => {
debug!(target: "libp2p-core", "error while polling listener: {:?}", err);
debug!("error while polling listener: {:?}", err);
if self.connections.is_empty() && self.current_upgrades.is_empty() {
return Err(err);
}
@ -296,8 +296,7 @@ where
}
Err(err) => {
// Insert the rest of the pending upgrades, but not the current one.
debug!(target: "libp2p-core", "error while upgrading listener connection: \
{:?}", err);
debug!("error while upgrading listener connection: {:?}", err);
return Ok(Async::Ready(Some(future::err(err))));
}
_ => {}
@ -309,7 +308,7 @@ where
match next_incoming.poll() {
Ok(Async::Ready(None)) => {
// stream muxer gave us a `None` => connection should be considered closed
debug!(target: "libp2p-core", "no more inbound substreams on {}", client_addr);
debug!("no more inbound substreams on {}", client_addr);
self.shared.lock().active_connections.remove(&client_addr);
}
Ok(Async::Ready(Some(incoming))) => {
@ -331,8 +330,7 @@ where
self.connections.push((muxer, next_incoming, client_addr));
}
Err(err) => {
debug!(target: "libp2p-core", "error while upgrading the \
multiplexed incoming connection: {:?}", err);
debug!("error while upgrading the multiplexed incoming connection: {:?}", err);
// Insert the rest of the pending connections, but not the current one.
return Ok(Async::Ready(Some(future::err(err))));
}
@ -384,13 +382,13 @@ where
let (muxer, mut future, addr) = lock.next_incoming.swap_remove(n);
match future.poll() {
Ok(Async::Ready(None)) => {
debug!(target: "libp2p-core", "no inbound substream for {}", addr);
debug!("no inbound substream for {}", addr);
lock.active_connections.remove(&addr);
}
Ok(Async::Ready(Some(value))) => {
// A substream is ready ; push back the muxer for the next time this function
// is called, then return.
debug!(target: "libp2p-core", "New incoming substream");
debug!("New incoming substream");
let next = muxer.clone().inbound();
lock.next_incoming.push((muxer, next, addr.clone()));
return Ok(Async::Ready(future::ok((value, addr))));
@ -400,9 +398,9 @@ where
}
Err(err) => {
// In case of error, we just not push back the element, which drops it.
debug!(target: "libp2p-core", "ConnectionReuse incoming: one of the \
multiplexed substreams produced an error: {:?}",
err);
debug!("ConnectionReuse incoming: one of the \
multiplexed substreams produced an error: {:?}",
err);
}
}
}

View File

@ -118,7 +118,7 @@ where
Du: Transport + 'static, // TODO: 'static :-/
Du::Output: Into<T::Output>,
{
trace!(target: "libp2p-core", "Swarm dialing {}", multiaddr);
trace!("Swarm dialing {}", multiaddr);
match transport.dial(multiaddr.clone()) {
Ok(dial) => {
@ -151,7 +151,7 @@ where
Df: FnOnce(Du::Output, Multiaddr) -> Dfu + 'static, // TODO: 'static :-/
Dfu: IntoFuture<Item = (), Error = IoError> + 'static, // TODO: 'static :-/
{
trace!(target: "libp2p-core", "Swarm dialing {} with custom handler", multiaddr);
trace!("Swarm dialing {} with custom handler", multiaddr);
match transport.dial(multiaddr) {
Ok(dial) => {
@ -170,7 +170,7 @@ where
pub fn listen_on(&self, multiaddr: Multiaddr) -> Result<Multiaddr, Multiaddr> {
match self.transport.clone().listen_on(multiaddr) {
Ok((listener, new_addr)) => {
trace!(target: "libp2p-core", "Swarm listening on {}", new_addr);
trace!("Swarm listening on {}", new_addr);
// Ignoring errors if the receiver has been closed, because in that situation
// nothing is going to be processed anyway.
let _ = self.new_listeners.unbounded_send(listener);
@ -224,15 +224,13 @@ where
match self.next_incoming.poll() {
Ok(Async::Ready(connec)) => {
debug!(target: "libp2p-core", "Swarm received new multiplexed \
incoming connection");
debug!("Swarm received new multiplexed incoming connection");
self.next_incoming = self.transport.clone().next_incoming();
self.listeners_upgrade.push(Box::new(connec) as Box<_>);
}
Ok(Async::NotReady) => {}
Err(err) => {
debug!(target: "libp2p-core", "Error in multiplexed incoming \
connection: {:?}", err);
debug!("Error in multiplexed incoming connection: {:?}", err);
self.next_incoming = self.transport.clone().next_incoming();
}
};
@ -272,12 +270,12 @@ where
match self.listeners.poll() {
Ok(Async::Ready(Some((Some(upgrade), remaining)))) => {
trace!(target: "libp2p-core", "Swarm received new connection on listener socket");
trace!("Swarm received new connection on listener socket");
self.listeners_upgrade.push(upgrade);
self.listeners.push(remaining.into_future());
}
Err((err, _)) => {
warn!(target: "libp2p-core", "Error in listener: {:?}", err);
warn!("Error in listener: {:?}", err);
}
_ => {}
}
@ -293,7 +291,7 @@ where
));
}
Err(err) => {
warn!(target: "libp2p-core", "Error in listener upgrade: {:?}", err);
warn!("Error in listener upgrade: {:?}", err);
}
_ => {}
}
@ -305,17 +303,17 @@ where
.push(future::Either::A(handler(output, addr).into_future()));
}
Err(err) => {
warn!(target: "libp2p-core", "Error in dialer upgrade: {:?}", err);
warn!("Error in dialer upgrade: {:?}", err);
}
_ => {}
}
match self.to_process.poll() {
Ok(Async::Ready(Some(()))) => {
trace!(target: "libp2p-core", "Future returned by swarm handler driven to completion");
trace!("Future returned by swarm handler driven to completion");
}
Err(err) => {
warn!(target: "libp2p-core", "Error in processing: {:?}", err);
warn!("Error in processing: {:?}", err);
}
_ => {}
}

View File

@ -45,7 +45,7 @@ where
.protocol_names()
.map::<_, fn(_) -> _>(|(n, t)| (n, <Bytes as PartialEq>::eq, t));
let remote_addr2 = remote_addr.clone();
debug!(target: "libp2p-core", "Starting protocol negotiation");
debug!("Starting protocol negotiation");
let negotiation = match endpoint {
Endpoint::Listener => multistream_select::listener_select_proto(connection, iter),
@ -56,10 +56,8 @@ where
.map_err(|err| IoError::new(IoErrorKind::Other, err))
.then(move |negotiated| {
match negotiated {
Ok(_) => debug!(target: "libp2p-core", "Successfully negotiated \
protocol upgrade with {}", remote_addr2),
Err(ref err) => debug!(target: "libp2p-core", "Error while negotiated \
protocol upgrade: {:?}", err),
Ok(_) => debug!("Successfully negotiated protocol upgrade with {}", remote_addr2),
Err(ref err) => debug!("Error while negotiated protocol upgrade: {:?}", err),
};
negotiated
})
@ -70,9 +68,8 @@ where
.into_future()
.then(|val| {
match val {
Ok(_) => debug!(target: "libp2p-core", "Successfully applied negotiated \
protocol"),
Err(_) => debug!(target: "libp2p-core", "Failed to apply negotiated protocol"),
Ok(_) => debug!("Successfully applied negotiated protocol"),
Err(_) => debug!("Failed to apply negotiated protocol"),
}
val
});

View File

@ -44,7 +44,7 @@ extern crate multiaddr;
extern crate tokio_dns;
extern crate tokio_io;
use futures::future::{self, Future, IntoFuture};
use futures::future::{self, Future};
use log::Level;
use multiaddr::{AddrComponent, Multiaddr};
use std::fmt;
@ -76,7 +76,7 @@ impl<T> DnsConfig<T> {
/// Same as `new`, but allows specifying a number of threads for the resolving.
#[inline]
pub fn with_resolve_threads(inner: T, num_threads: usize) -> DnsConfig<T> {
trace!(target: "libp2p-dns", "Created a CpuPoolResolver");
trace!("Created a CpuPoolResolver");
DnsConfig {
inner,
@ -126,7 +126,7 @@ where
});
if !contains_dns {
trace!(target: "libp2p-dns", "Pass-through address without DNS: {}", addr);
trace!("Pass-through address without DNS: {}", addr);
return match self.inner.dial(addr) {
Ok(d) => Ok(Box::new(d) as Box<_>),
Err((inner, addr)) => Err((
@ -141,7 +141,7 @@ where
let resolver = self.resolver;
trace!(target: "libp2p-dns", "Dialing address with DNS: {}", addr);
trace!("Dialing address with DNS: {}", addr);
let resolve_iters = addr.iter()
.map(move |cmp| match cmp {
AddrComponent::DNS4(ref name) => {
@ -157,7 +157,7 @@ where
let new_addr = future::join_all(resolve_iters).map(move |outcome| {
let outcome: Multiaddr = outcome.into_iter().collect();
debug!(target: "libp2p-dns", "DNS resolution outcome: {} => {}", addr, outcome);
debug!("DNS resolution outcome: {} => {}", addr, outcome);
outcome
});
@ -194,15 +194,15 @@ fn resolve_dns(
resolver: CpuPoolResolver,
ty: ResolveTy,
) -> Box<Future<Item = AddrComponent, Error = IoError>> {
let debug_name = if log_enabled!(target: "libp2p-dns", Level::Trace) {
let debug_name = if log_enabled!(Level::Trace) {
Some(name.to_owned())
} else {
None
};
let future = resolver.resolve(name).and_then(move |addrs| {
if log_enabled!(target: "libp2p-dns", Level::Trace) {
trace!(target: "libp2p-dns", "DNS component resolution: {} => {:?}",
if log_enabled!(Level::Trace) {
trace!("DNS component resolution: {} => {:?}",
debug_name.expect("trace log level was enabled"), addrs);
}

View File

@ -112,7 +112,7 @@ where
_: Endpoint,
remote_addr: &Multiaddr,
) -> Self::Future {
debug!(target: "libp2p-floodsub", "Upgrading connection to {} as floodsub", remote_addr);
debug!("Upgrading connection to {} as floodsub", remote_addr);
// Whenever a new node connects, we send to it a message containing the topics we are
// already subscribed to.
@ -192,8 +192,7 @@ where
Some((bytes, MessageSource::FromChannel)) => {
// Received a packet from the channel.
// Need to send a message to remote.
trace!(target: "libp2p-floodsub", "Effectively sending message \
to remote");
trace!("Effectively sending message to remote");
let future = floodsub_sink.send(bytes).map(|floodsub_sink| {
future::Loop::Continue((floodsub_sink, rest))
});
@ -203,7 +202,7 @@ where
None => {
// Both the connection stream and `rx` are empty, so we break
// the loop.
trace!(target: "libp2p-floodsub", "Pubsub future clean finish");
trace!("Pubsub future clean finish");
// TODO: what if multiple connections?
inner.remote_connections.write().remove(&remote_addr);
let future = future::ok(future::Loop::Break(()));
@ -341,9 +340,8 @@ impl FloodSubController {
let topics = topics.into_iter();
if log_enabled!(target: "libp2p-floodsub", Level::Debug) {
debug!(target: "libp2p-floodsub", "Queuing sub/unsub message ; \
sub = {:?} ; unsub = {:?}",
if log_enabled!(Level::Debug) {
debug!("Queuing sub/unsub message ; sub = {:?} ; unsub = {:?}",
topics.clone().filter(|t| t.1)
.map(|t| t.0.hash().clone().into_string())
.collect::<Vec<_>>(),
@ -387,8 +385,7 @@ impl FloodSubController {
{
let topics = topics.into_iter().collect::<Vec<_>>();
debug!(target: "libp2p-floodsub", "Queueing publish message ; \
topics = {:?} ; data_len = {:?}",
debug!("Queueing publish message ; topics = {:?} ; data_len = {:?}",
topics.iter().map(|t| t.hash().clone().into_string()).collect::<Vec<_>>(),
data.len());
@ -457,8 +454,8 @@ impl FloodSubController {
match remote.sender.unbounded_send(bytes.clone().into()) {
Ok(_) => (),
Err(_) => {
trace!(target: "libp2p-floodsub", "Failed to dispatch message to {} because \
channel was closed", remote_addr);
trace!("Failed to dispatch message to {} because channel was closed",
remote_addr);
failed_to_send.push(remote_addr.clone());
}
}
@ -474,7 +471,7 @@ impl FloodSubController {
}
}
debug!(target: "libp2p-floodsub", "Message queued for {} remotes", num_dispatched);
debug!("Message queued for {} remotes", num_dispatched);
}
}
@ -546,13 +543,13 @@ fn handle_packet_received(
inner: Arc<Inner>,
remote_addr: &Multiaddr,
) -> Result<(), IoError> {
trace!(target: "libp2p-floodsub", "Received packet from {}", remote_addr);
trace!("Received packet from {}", remote_addr);
// Parsing attempt.
let mut input = match protobuf::parse_from_bytes::<rpc_proto::RPC>(&bytes) {
Ok(msg) => msg,
Err(err) => {
debug!(target: "libp2p-floodsub", "Failed to parse protobuf message ; err = {:?}", err);
debug!("Failed to parse protobuf message ; err = {:?}", err);
return Err(err.into());
}
};
@ -567,12 +564,9 @@ fn handle_packet_received(
let topic = TopicHash::from_raw(subscription.take_topicid());
let subscribe = subscription.get_subscribe();
if subscribe {
trace!(target: "libp2p-floodsub", "Remote {} subscribed to {:?}",
remote_addr, topic);
topics.insert(topic);
trace!("Remote {} subscribed to {:?}", remote_addr, topic); topics.insert(topic);
} else {
trace!(target: "libp2p-floodsub", "Remote {} unsubscribed from {:?}",
remote_addr, topic);
trace!("Remote {} unsubscribed from {:?}", remote_addr, topic);
topics.remove(&topic);
}
}
@ -589,8 +583,7 @@ fn handle_packet_received(
.lock()
.insert(hash((from.clone(), publish.take_seqno())))
{
trace!(target: "libp2p-floodsub",
"Skipping message because we had already received it ; payload = {} bytes",
trace!("Skipping message because we had already received it ; payload = {} bytes",
publish.get_data().len());
continue;
}
@ -602,9 +595,9 @@ fn handle_packet_received(
.map(|h| TopicHash::from_raw(h))
.collect::<Vec<_>>();
trace!(target: "libp2p-floodsub",
"Processing message for topics {:?} ; payload = {} bytes",
topics, publish.get_data().len());
trace!("Processing message for topics {:?} ; payload = {} bytes",
topics,
publish.get_data().len());
// TODO: should check encryption/authentication of the message
@ -617,8 +610,7 @@ fn handle_packet_received(
continue;
}
// TODO: don't send back to the remote that just sent it
trace!(target: "libp2p-floodsub",
"Broadcasting received message to {}", addr);
trace!("Broadcasting received message to {}", addr);
let _ = info.sender.unbounded_send(bytes.clone());
}
}
@ -632,15 +624,14 @@ fn handle_packet_received(
};
if dispatch_locally {
// Ignore if channel is closed.
trace!(target: "libp2p-floodsub", "Dispatching message locally");
trace!("Dispatching message locally");
let _ = inner.output_tx.unbounded_send(Message {
source: from,
data: publish.take_data(),
topics: topics,
});
} else {
trace!(target: "libp2p-floodsub",
"Message not dispatched locally as we are not subscribed to any of the topics");
trace!("Message not dispatched locally as we are not subscribed to any of the topics");
}
}

View File

@ -72,8 +72,8 @@ where
info: IdentifyInfo,
observed_addr: &Multiaddr,
) -> Box<Future<Item = (), Error = IoError> + 'a> {
debug!(target: "libp2p-identify", "Sending identify info to client");
trace!(target: "libp2p-identify", "Sending: {:?}", info);
debug!("Sending identify info to client");
trace!("Sending: {:?}", info);
let listen_addrs = info.listen_addrs
.into_iter()
@ -128,11 +128,10 @@ where
}
fn upgrade(self, socket: C, _: (), ty: Endpoint, observed_addr: &Multiaddr) -> Self::Future {
trace!(target: "libp2p-identify", "Upgrading connection with {:?} as {:?}",
observed_addr, ty);
trace!("Upgrading connection with {:?} as {:?}", observed_addr, ty);
let socket = socket.framed(VarintCodec::default());
let observed_addr_log = if log_enabled!(target: "libp2p-identify", Level::Debug) {
let observed_addr_log = if log_enabled!(Level::Debug) {
Some(observed_addr.clone())
} else {
None
@ -145,8 +144,8 @@ where
.map(|(msg, _)| msg)
.map_err(|(err, _)| err)
.and_then(|msg| {
if log_enabled!(target: "libp2p-identify", Level::Debug) {
debug!(target: "libp2p-identify", "Received identify message from {:?}",
if log_enabled!(Level::Debug) {
debug!("Received identify message from {:?}",
observed_addr_log
.expect("Programmer error: expected `observed_addr_log' to be \
non-None since debug log level is enabled"));
@ -156,23 +155,20 @@ where
let (info, observed_addr) = match parse_proto_msg(msg) {
Ok(v) => v,
Err(err) => {
debug!(target: "libp2p-identify",
"Failed to parse protobuf message ; error = {:?}", err);
debug!("Failed to parse protobuf message ; error = {:?}", err);
return Err(err.into());
}
};
trace!(target: "libp2p-identify", "Remote observes us as {:?}",
observed_addr);
trace!(target: "libp2p-identify", "Information received: {:?}", info);
trace!("Remote observes us as {:?}", observed_addr);
trace!("Information received: {:?}", info);
Ok(IdentifyOutput::RemoteInfo {
info,
observed_addr,
})
} else {
debug!(target: "libp2p-identify", "Identify protocol stream closed \
before receiving info");
debug!("Identify protocol stream closed before receiving info");
Err(IoErrorKind::InvalidData.into())
}
});

View File

@ -101,17 +101,14 @@ where
};
if peer.addrs().any(|addr| addr == client_addr) {
debug!(target: "libp2p-identify", "Incoming substream from {} \
identified as {:?}", client_addr,
peer_id);
debug!("Incoming substream from {} identified as {:?}", client_addr, peer_id);
let out = IdentifyTransportOutput { socket: connec, observed_addr: None };
let ret = (out, AddrComponent::P2P(peer_id.into_bytes()).into());
return future::Either::A(future::ok(ret));
}
}
debug!(target: "libp2p-identify", "Incoming connection from {}, dialing back \
in order to identify", client_addr);
debug!("Incoming connection from {}, dialing back in order to identify", client_addr);
// Dial the address that connected to us and try upgrade with the
// identify protocol.
let future = identify_upgrade
@ -143,14 +140,12 @@ where
),
};
debug!(target: "libp2p-identify", "Identified {} as {}", original_addr,
real_addr);
debug!("Identified {} as {}", original_addr, real_addr);
let out = IdentifyTransportOutput { socket: connec, observed_addr: Some(observed) };
Ok((out, real_addr))
})
.map_err(move |err| {
warn!(target: "libp2p-identify", "Failed to identify incoming {}",
client_addr);
warn!("Failed to identify incoming {}", client_addr);
err
});
future::Either::B(future)
@ -176,8 +171,7 @@ where
.collect::<Vec<_>>()
.into_iter();
trace!(target: "libp2p-identify", "Try dialing peer ID {:?} ; {} multiaddrs \
loaded from peerstore", peer_id, addrs.len());
trace!("Try dialing peer ID {:?} ; {} multiaddrs loaded from peerstore", peer_id, addrs.len());
let transport = self.transport;
let future = stream::iter_ok(addrs)
@ -186,8 +180,7 @@ where
match transport.clone().dial(addr) {
Ok(dial) => Some(dial),
Err((_, addr)) => {
warn!(target: "libp2p-identify", "Address {} not supported by \
underlying transport", addr);
warn!("Address {} not supported by underlying transport", addr);
None
},
}
@ -201,15 +194,12 @@ where
.and_then(move |(val, _)| {
match val {
Some((connec, inner_addr)) => {
debug!(target: "libp2p-identify", "Successfully dialed peer {:?} \
through {}", peer_id,
inner_addr);
debug!("Successfully dialed peer {:?} through {}", peer_id, inner_addr);
let out = IdentifyTransportOutput { socket: connec, observed_addr: None };
Ok((out, inner_addr))
},
None => {
debug!(target: "libp2p-identify", "All multiaddresses failed when \
dialing peer {:?}", peer_id);
debug!("All multiaddresses failed when dialing peer {:?}", peer_id);
// TODO: wrong error
Err(IoErrorKind::InvalidData.into())
},
@ -227,7 +217,7 @@ where
let transport = self.transport;
let identify_upgrade = transport.clone().with_upgrade(IdentifyProtocolConfig);
trace!(target: "libp2p-identify", "Pass through when dialing {}", addr);
trace!("Pass through when dialing {}", addr);
// We dial a first time the node and upgrade it to identify.
let dial = match identify_upgrade.dial(addr) {
@ -311,9 +301,7 @@ where
};
if peer.addrs().any(|addr| addr == client_addr) {
debug!(target: "libp2p-identify", "Incoming substream from {} \
identified as {:?}", client_addr,
peer_id);
debug!("Incoming substream from {} identified as {:?}", client_addr, peer_id);
let out = IdentifyTransportOutput { socket: connec, observed_addr: None };
let ret = (out, AddrComponent::P2P(peer_id.into_bytes()).into());
return future::Either::A(future::ok(ret));

View File

@ -142,8 +142,7 @@ fn query<'a, I>(
where
I: QueryInterface + 'a,
{
debug!(target: "libp2p-kad", "Start query for {:?} ; num results = {}", searched_key,
num_results);
debug!("Start query for {:?} ; num results = {}", searched_key, num_results);
// State of the current iterative process.
struct State<'a> {
@ -209,9 +208,9 @@ where
to_contact
};
debug!(target: "libp2p-kad", "New query round ; {} queries in progress ; contacting \
{} new peers", state.current_attempts_fut.len(),
to_contact.len());
debug!("New query round ; {} queries in progress ; contacting {} new peers",
state.current_attempts_fut.len(),
to_contact.len());
// For each node in `to_contact`, start an RPC query and a corresponding entry in the two
// `state.current_attempts_*` fields.
@ -262,7 +261,7 @@ where
let closer_peers = match message {
Ok(msg) => msg,
Err(err) => {
trace!(target: "libp2p-kad", "RPC query failed for {:?}: {:?}", remote_id, err);
trace!("RPC query failed for {:?}: {:?}", remote_id, err);
state.failed_to_contact.insert(remote_id);
return Ok(future::Loop::Continue(state));
}
@ -294,8 +293,7 @@ where
// the remote.
{
let valid_multiaddrs = peer.multiaddrs.drain(..);
trace!(target: "libp2p-kad", "Adding multiaddresses to {:?}: {:?}",
peer.node_id, valid_multiaddrs);
trace!("Adding multiaddresses to {:?}: {:?}", peer.node_id, valid_multiaddrs);
query_interface2.peer_add_addrs(
&peer.node_id,
valid_multiaddrs,
@ -334,7 +332,7 @@ where
Ok(future::Loop::Break(state))
} else {
if !local_nearest_node_updated {
trace!(target: "libp2p-kad", "Loop didn't update closer node ; jumping to step 2");
trace!("Loop didn't update closer node ; jumping to step 2");
state.looking_for_closer = false;
}
@ -346,7 +344,7 @@ where
});
let stream = stream.map(|state| {
debug!(target: "libp2p-kad", "Query finished with {} results", state.result.len());
debug!("Query finished with {} results", state.result.len());
state.result
});

View File

@ -243,7 +243,7 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
substream_id,
packet_type,
} = MultiplexHeader::parse(header).map_err(|err| {
debug!(target: "libp2p-mplex", "failed to parse header: {}", err);
debug!("failed to parse header: {}", err);
io::Error::new(
io::ErrorKind::Other,
format!("Error parsing header: {:?}", err),
@ -282,10 +282,10 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
}
Err(error) => {
return if let varint::Error(varint::ErrorKind::Io(inner), ..) = error {
debug!(target: "libp2p-mplex", "failed to read header: {}", inner);
debug!("failed to read header: {}", inner);
Err(inner)
} else {
debug!(target: "libp2p-mplex", "failed to read header: {}", error);
debug!("failed to read header: {}", error);
Err(io::Error::new(io::ErrorKind::Other, error.description()))
};
}
@ -298,10 +298,7 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
use self::NextMultiplexState::*;
let body_len = varint_state.read(&mut lock.stream).map_err(|e| {
debug!(target: "libp2p-mplex",
"substream {}: failed to read body length: {}",
next.substream_id(),
e);
debug!("substream {}: failed to read body length: {}", next.substream_id(), e);
io::Error::new(io::ErrorKind::Other, "Error reading varint")
})?;
@ -401,8 +398,7 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
return on_block;
}
Err(other) => {
debug!(target: "libp2p-mplex",
"substream {}: failed to read new stream: {}",
debug!("substream {}: failed to read new stream: {}",
substream_id,
other);
lock.read_state = Some(NewStream {
@ -469,8 +465,7 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
return on_block;
}
Err(other) => {
debug!(target: "libp2p-mplex",
"substream {}: failed to read message body: {}",
debug!("substream {}: failed to read message body: {}",
substream_id,
other);
return Err(other);
@ -520,8 +515,7 @@ fn read_stream_internal<T: AsyncRead, Buf: Array<Item = u8>>(
return on_block;
}
Err(other) => {
debug!(target: "libp2p-mplex",
"substream {}: failed to read ignore bytes: {}",
debug!("substream {}: failed to read ignore bytes: {}",
substream_id,
other);
lock.read_state = Some(Ignore {

View File

@ -98,7 +98,7 @@ impl<T, Buf: Array> MultiplexShared<T, Buf> {
}
pub fn open_stream(&mut self, id: u32) -> bool {
trace!(target: "libp2p-mplex", "open stream {}", id);
trace!("open stream {}", id);
self.open_streams
.entry(id)
.or_insert(SubstreamMetadata::new_open())
@ -106,7 +106,7 @@ impl<T, Buf: Array> MultiplexShared<T, Buf> {
}
pub fn close_stream(&mut self, id: u32) {
trace!(target: "libp2p-mplex", "close stream {}", id);
trace!("close stream {}", id);
self.open_streams.insert(id, SubstreamMetadata::Closed);
}

View File

@ -207,10 +207,7 @@ pub fn write_stream<Buf: circular_buffer::Array, T: AsyncWrite>(
Err(Some(Body { size }))
}
Err(other) => {
debug!(target: "libp2p-mplex",
"substream {}: failed to write body: {}",
id,
other);
debug!("substream {}: failed to write body: {}", id, other);
return Err(other);
}
}

View File

@ -85,7 +85,7 @@ where
let req = DialerToListenerMessage::ProtocolRequest {
name: proto_name.clone()
};
trace!(target: "multistream-select", "sending {:?}", req);
trace!("sending {:?}", req);
dialer.send(req)
.map(|d| (d, proto_name, proto_value))
.from_err()
@ -99,7 +99,7 @@ where
})
// Once read, analyze the response.
.and_then(|(message, rest, proto_name, proto_value)| {
trace!(target: "multistream-select", "received {:?}", message);
trace!("received {:?}", message);
let message = message.ok_or(ProtocolChoiceError::UnexpectedMessage)?;
match message {
@ -140,14 +140,14 @@ where
let future = Dialer::new(inner)
.from_err()
.and_then(move |dialer| {
trace!(target: "multistream-select", "requesting protocols list");
trace!("requesting protocols list");
dialer
.send(DialerToListenerMessage::ProtocolsListRequest)
.from_err()
})
.and_then(move |dialer| dialer.into_future().map_err(|(e, _)| e.into()))
.and_then(move |(msg, dialer)| {
trace!(target: "multistream-select", "protocols list response: {:?}", msg);
trace!("protocols list response: {:?}", msg);
let list = match msg {
Some(ListenerToDialerMessage::ProtocolsListResponse { list }) => list,
_ => return Err(ProtocolChoiceError::UnexpectedMessage),
@ -171,7 +171,7 @@ where
Ok((proto_name, proto_val, dialer))
})
.and_then(|(proto_name, proto_val, dialer)| {
trace!(target: "multistream-select", "sending {:?}", proto_name);
trace!("sending {:?}", proto_name);
dialer
.send(DialerToListenerMessage::ProtocolRequest {
name: proto_name.clone(),
@ -186,7 +186,7 @@ where
.map_err(|(err, _)| err.into())
})
.and_then(|(proto_name, proto_val, msg, dialer)| {
trace!(target: "multistream-select", "received {:?}", msg);
trace!("received {:?}", msg);
match msg {
Some(ListenerToDialerMessage::ProtocolAck { ref name }) if name == &proto_name => {
Ok((proto_val, dialer.into_inner()))

View File

@ -68,7 +68,7 @@ where
let msg = ListenerToDialerMessage::ProtocolsListResponse {
list: protocols.map(|(p, _, _)| p).collect(),
};
trace!(target: "multistream-select", "protocols list response: {:?}", msg);
trace!("protocols list response: {:?}", msg);
let fut = listener
.send(msg)
.from_err()
@ -86,10 +86,7 @@ where
break;
}
}
trace!(target: "multistream-select",
"requested: {:?}, response: {:?}",
name,
send_back);
trace!("requested: {:?}, response: {:?}", name, send_back);
let fut = listener
.send(send_back)
.from_err()
@ -97,7 +94,7 @@ where
Box::new(fut) as Box<Future<Item = _, Error = ProtocolChoiceError>>
}
None => {
debug!(target: "multistream-select", "no protocol request received");
debug!("no protocol request received");
Box::new(err(ProtocolChoiceError::NoProtocolFound)) as Box<_>
}
})

View File

@ -58,7 +58,7 @@ where
.map_err(|(e, _)| e.into())
.and_then(|(msg, rest)| {
if msg.as_ref().map(|b| &b[..]) != Some(MULTISTREAM_PROTOCOL_WITH_LF) {
debug!(target: "multistream-select", "failed handshake; received: {:?}", msg);
debug!("failed handshake; received: {:?}", msg);
return Err(MultistreamSelectError::FailedHandshake);
}
Ok(rest)
@ -93,7 +93,7 @@ where
match item {
ListenerToDialerMessage::ProtocolAck { name } => {
if !name.starts_with(b"/") {
debug!(target: "multistream-select", "invalid protocol name {:?}", name);
debug!("invalid protocol name {:?}", name);
return Err(MultistreamSelectError::WrongProtocolName);
}
let mut protocol = BytesMut::from(name);

View File

@ -165,7 +165,7 @@ where
.map(|msg| Message::Received(msg.freeze()));
let (sink, stream) = sink_stream.split();
let remote_addr = if log_enabled!(target: "libp2p-ping", Level::Debug) {
let remote_addr = if log_enabled!(Level::Debug) {
Some(remote_addr.clone())
} else {
None
@ -185,9 +185,8 @@ where
match message {
Message::Ping(payload, finished) => {
// Ping requested by the user through the `Pinger`.
if log_enabled!(target: "libp2p-ping", Level::Debug) {
debug!(target: "libp2p-ping", "Sending ping to {:?} with \
payload {:?}",
if log_enabled!(Level::Debug) {
debug!("Sending ping to {:?} with payload {:?}",
remote_addr.expect("debug log level is enabled"),
payload);
}
@ -205,16 +204,14 @@ where
// Payload was ours. Signalling future.
// Errors can happen if the user closed the receiving end of
// the future, which is fine to ignore.
debug!(target: "libp2p-ping", "Received pong from {:?} \
(payload={:?}) ; ping fufilled",
debug!("Received pong from {:?} (payload={:?}) ; ping fufilled",
remote_addr.expect("debug log level is enabled"), payload);
let _ = fut.send(());
Box::new(Ok(Loop::Continue((sink, stream))).into_future())
as Box<Future<Item = _, Error = _>>
} else {
// Payload was not ours. Sending it back.
debug!(target: "libp2p-ping", "Received ping from {:?} \
(payload={:?}) ; sending back",
debug!("Received ping from {:?} (payload={:?}) ; sending back",
remote_addr.expect("debug log level is enabled"), payload);
Box::new(
sink.send(payload)
@ -249,7 +246,7 @@ impl Pinger {
pub fn ping(&mut self) -> Box<Future<Item = (), Error = Box<Error + Send + Sync>>> {
let (tx, rx) = oneshot::channel();
let payload: [u8; 32] = Rand::rand(&mut self.os_rng);
debug!(target: "libp2p-ping", "Preparing for ping with payload {:?}", payload);
debug!("Preparing for ping with payload {:?}", payload);
// Ignore errors if the ponger has been already destroyed. The returned future will never
// be signalled.
let fut = self.send

View File

@ -51,11 +51,11 @@ impl<T> RateLimited<T> {
Ok(RateLimited {
value,
rlimiter: Limiter::new(e, max_read).map_err(|e| {
error!(target: "libp2p-ratelimit", "failed to create read limiter: {}", e);
error!("failed to create read limiter: {}", e);
io::Error::new(io::ErrorKind::Other, e)
})?,
wlimiter: Limiter::new(e, max_write).map_err(|e| {
error!(target: "libp2p-ratelimit", "failed to create write limiter: {}", e);
error!("failed to create write limiter: {}", e);
io::Error::new(io::ErrorKind::Other, e)
})?,
})
@ -81,11 +81,11 @@ impl<C: AsyncRead + AsyncWrite> Connection<C> {
let (r, w) = c.split();
Ok(Connection {
reader: Limited::new(r, rlimiter).map_err(|e| {
error!(target: "libp2p-ratelimit", "failed to create limited reader: {}", e);
error!("failed to create limited reader: {}", e);
io::Error::new(io::ErrorKind::Other, e)
})?,
writer: Limited::new(w, wlimiter).map_err(|e| {
error!(target: "libp2p-ratelimit", "failed to create limited writer: {}", e);
error!("failed to create limited writer: {}", e);
io::Error::new(io::ErrorKind::Other, e)
})?,
})

View File

@ -80,7 +80,7 @@ where
let hmac_num_bytes = self.hmac_key.digest_algorithm().output_len;
if frame.len() < hmac_num_bytes {
debug!(target: "libp2p-secio", "frame too short when decoding secio frame");
debug!("frame too short when decoding secio frame");
return Err(SecioError::FrameTooShort);
}
@ -88,7 +88,7 @@ where
debug_assert_eq!(expected_hash.len(), hmac_num_bytes);
if let Err(_) = hmac::verify(&self.hmac_key, crypted_data, expected_hash) {
debug!(target: "libp2p-secio", "hmac mismatch when decoding secio frame");
debug!("hmac mismatch when decoding secio frame");
return Err(SecioError::HmacNotMatching);
}

View File

@ -139,8 +139,7 @@ where
.and_then(|mut context| {
context.rng.fill(&mut context.local_nonce)
.map_err(|_| SecioError::NonceGenerationFailed)?;
trace!(target: "libp2p-secio", "starting handshake ; local pubkey = {:?} ; \
local nonce = {:?}",
trace!("starting handshake ; local pubkey = {:?} ; local nonce = {:?}",
context.local_public_key, context.local_nonce);
Ok(context)
})
@ -161,7 +160,7 @@ where
let proposition_bytes = proposition.write_to_bytes().unwrap();
context.local_proposition_bytes = proposition_bytes.clone();
trace!(target: "libp2p-secio", "sending proposition to remote");
trace!("sending proposition to remote");
socket.send(BytesMut::from(proposition_bytes.clone()))
.from_err()
@ -177,8 +176,7 @@ where
Some(p) => context.remote_proposition_bytes = p,
None => {
let err = IoError::new(IoErrorKind::BrokenPipe, "unexpected eof");
debug!(target: "libp2p-secio", "unexpected eof while waiting for \
remote's proposition");
debug!("unexpected eof while waiting for remote's proposition");
return Err(err.into())
},
};
@ -188,8 +186,7 @@ where
) {
Ok(prop) => prop,
Err(_) => {
debug!(target: "libp2p-secio", "failed to parse remote's proposition \
protobuf message");
debug!("failed to parse remote's proposition protobuf message");
return Err(SecioError::HandshakeParsingFailure);
}
};
@ -199,8 +196,7 @@ where
match protobuf_parse_from_bytes::<PublicKeyProtobuf>(bytes) {
Ok(p) => p,
Err(_) => {
debug!(target: "libp2p-secio", "failed to parse remote's \
proposition's pubkey protobuf");
debug!("failed to parse remote's proposition's pubkey protobuf");
return Err(SecioError::HandshakeParsingFailure);
},
}
@ -212,15 +208,13 @@ where
KeyTypeProtobuf::RSA => (),
format => {
let err = IoError::new(IoErrorKind::Other, "unsupported protocol");
debug!(target: "libp2p-secio", "unsupported remote pubkey format {:?}",
format);
debug!("unsupported remote pubkey format {:?}", format);
return Err(err.into());
},
};
context.remote_public_key = pubkey.take_Data();
context.remote_nonce = prop.take_rand();
trace!(target: "libp2p-secio", "received proposition from remote ; \
pubkey = {:?} ; nonce = {:?}",
trace!("received proposition from remote ; pubkey = {:?} ; nonce = {:?}",
context.remote_public_key, context.remote_nonce);
Ok((prop, socket, context))
})
@ -253,7 +247,7 @@ where
Some(match algo_support::exchanges::select_best(context.hashes_ordering, list) {
Ok(a) => a,
Err(err) => {
debug!(target: "libp2p-secio", "failed to select an exchange protocol");
debug!("failed to select an exchange protocol");
return Err(err);
}
})
@ -263,7 +257,7 @@ where
Some(match algo_support::ciphers::select_best(context.hashes_ordering, list) {
Ok(a) => a,
Err(err) => {
debug!(target: "libp2p-secio", "failed to select a cipher protocol");
debug!("failed to select a cipher protocol");
return Err(err);
}
})
@ -273,7 +267,7 @@ where
Some(match algo_support::hashes::select_best(context.hashes_ordering, list) {
Ok(a) => a,
Err(err) => {
debug!(target: "libp2p-secio", "failed to select a hash protocol");
debug!("failed to select a hash protocol");
return Err(err);
}
})
@ -287,7 +281,7 @@ where
match EphemeralPrivateKey::generate(&agreement::ECDH_P256, &context.rng) {
Ok(tmp_priv_key) => Ok((socket, context, tmp_priv_key)),
Err(_) => {
debug!(target: "libp2p-secio", "failed to generate ECDH key");
debug!("failed to generate ECDH key");
Err(SecioError::EphemeralKeyGenerationFailed)
},
}
@ -311,7 +305,7 @@ where
let mut state = match RSASigningState::new(context.local_private_key.clone()) {
Ok(s) => s,
Err(_) => {
debug!(target: "libp2p-secio", "failed to sign local exchange");
debug!("failed to sign local exchange");
return Err(SecioError::SigningFailure);
},
};
@ -321,7 +315,7 @@ where
{
Ok(_) => (),
Err(_) => {
debug!(target: "libp2p-secio", "failed to sign local exchange");
debug!("failed to sign local exchange");
return Err(SecioError::SigningFailure);
},
};
@ -339,7 +333,7 @@ where
// Send our local `Exchange`.
.and_then(|(local_exch, socket, context)| {
trace!(target: "libp2p-secio", "sending exchange to remote");
trace!("sending exchange to remote");
socket.send(local_exch)
.from_err()
.map(|s| (s, context))
@ -354,8 +348,7 @@ where
Some(r) => r,
None => {
let err = IoError::new(IoErrorKind::BrokenPipe, "unexpected eof");
debug!(target: "libp2p-secio", "unexpected eof while waiting for \
remote's exchange");
debug!("unexpected eof while waiting for remote's exchange");
return Err(err.into())
},
};
@ -363,13 +356,12 @@ where
let remote_exch = match protobuf_parse_from_bytes::<Exchange>(&raw) {
Ok(e) => e,
Err(err) => {
debug!(target: "libp2p-secio", "failed to parse remote's exchange \
protobuf ; {:?}", err);
debug!("failed to parse remote's exchange protobuf ; {:?}", err);
return Err(SecioError::HandshakeParsingFailure);
}
};
trace!(target: "libp2p-secio", "received and decoded the remote's exchange");
trace!("received and decoded the remote's exchange");
Ok((remote_exch, socket, context))
})
})
@ -392,12 +384,12 @@ where
{
Ok(()) => (),
Err(_) => {
debug!(target: "libp2p-secio", "failed to verify the remote's signature");
debug!("failed to verify the remote's signature");
return Err(SecioError::SignatureVerificationFailed)
},
}
trace!(target: "libp2p-secio", "successfully verified the remote's signature");
trace!("successfully verified the remote's signature");
Ok((remote_exch, socket, context))
})
@ -455,7 +447,7 @@ where
match codec {
Ok(c) => Ok((c, context)),
Err(err) => {
debug!(target: "libp2p-secio", "failed to generate shared secret with remote");
debug!("failed to generate shared secret with remote");
return Err(err);
},
}
@ -464,7 +456,7 @@ where
// We send back their nonce to check if the connection works.
.and_then(|(codec, mut context)| {
let remote_nonce = mem::replace(&mut context.remote_nonce, Vec::new());
trace!(target: "libp2p-secio", "checking encryption by sending back remote's nonce");
trace!("checking encryption by sending back remote's nonce");
codec.send(BytesMut::from(remote_nonce))
.map(|s| (s, context))
.from_err()
@ -477,15 +469,15 @@ where
.and_then(move |(nonce, rest)| {
match nonce {
Some(ref n) if n == &context.local_nonce => {
trace!(target: "libp2p-secio", "secio handshake success");
trace!("secio handshake success");
Ok((rest, context.remote_public_key))
},
None => {
debug!(target: "libp2p-secio", "unexpected eof during nonce check");
debug!("unexpected eof during nonce check");
Err(IoError::new(IoErrorKind::BrokenPipe, "unexpected eof").into())
},
_ => {
debug!(target: "libp2p-secio", "failed nonce verification with remote");
debug!("failed nonce verification with remote");
Err(SecioError::NonceVerificationFailed)
}
}

View File

@ -211,7 +211,7 @@ where
_: libp2p_core::Endpoint,
remote_addr: &Multiaddr,
) -> Self::Future {
info!(target: "libp2p-secio", "starting secio upgrade with {:?}", remote_addr);
info!("starting secio upgrade with {:?}", remote_addr);
let fut = SecioMiddleware::handshake(incoming, self.key);
let wrapped = fut.map(|(stream_sink, pubkey)| {
@ -224,7 +224,7 @@ where
#[inline]
fn map_err(err: SecioError) -> IoError {
debug!(target: "libp2p-secio", "error during secio handshake {:?}", err);
debug!("error during secio handshake {:?}", err);
IoError::new(IoErrorKind::InvalidData, err)
}

View File

@ -111,7 +111,7 @@ impl Transport for TcpConfig {
Err(_) => addr,
};
debug!(target: "libp2p-tcp-transport", "Now listening on {}", new_addr);
debug!("Now listening on {}", new_addr);
let future = future::result(listener)
.map(|listener| {
@ -119,7 +119,7 @@ impl Transport for TcpConfig {
listener.incoming().map(|(sock, addr)| {
let addr = addr.to_multiaddr()
.expect("generating a multiaddr from a socket addr never fails");
debug!(target: "libp2p-tcp-transport", "Incoming connection from {}", addr);
debug!("Incoming connection from {}", addr);
Ok((sock, addr)).into_future()
})
})
@ -135,7 +135,7 @@ impl Transport for TcpConfig {
/// or gives back the multiaddress.
fn dial(self, addr: Multiaddr) -> Result<Self::Dial, (Self, Multiaddr)> {
if let Ok(socket_addr) = multiaddr_to_socketaddr(&addr) {
debug!(target: "libp2p-tcp-transport", "Dialing {}", addr);
debug!("Dialing {}", addr);
let fut = TcpStream::connect(&socket_addr, &self.event_loop).map(|t| (t, addr));
Ok(Box::new(fut) as Box<_>)
} else {

View File

@ -76,7 +76,7 @@ impl Transport for BrowserWsConfig {
Err(_) => return Err((self, original_addr)),
};
debug!(target: "libp2p-websocket", "Dialing {}", original_addr);
debug!("Dialing {}", original_addr);
// Create the JS `WebSocket` object.
let websocket = {

View File

@ -94,14 +94,14 @@ where
}
};
debug!(target: "libp2p-websocket", "Listening on {}", new_addr);
debug!("Listening on {}", new_addr);
let listen = inner_listen.map::<_, fn(_) -> _>(|stream| {
// Upgrade the listener to websockets like the websockets library requires us to do.
let upgraded = stream.and_then(|(stream, mut client_addr)| {
// Need to suffix `/ws` to each client address.
client_addr.append(AddrComponent::WS);
debug!(target: "libp2p-websocket", "Incoming connection from {}", client_addr);
debug!("Incoming connection from {}", client_addr);
stream
.into_ws()
@ -112,8 +112,7 @@ where
.accept()
.map_err(|err| IoError::new(IoErrorKind::Other, err))
.map(|(client, _http_headers)| {
debug!(target: "libp2p-websocket", "Upgraded incoming connection \
to websockets");
debug!("Upgraded incoming connection to websockets");
// Plug our own API on top of the `websockets` API.
let framed_data = client
@ -156,22 +155,22 @@ where
Some(AddrComponent::WS) => false,
Some(AddrComponent::WSS) => true,
_ => {
trace!(target: "libp2p-websocket", "Ignoring dial attempt for {} because it is \
not a websocket multiaddr", original_addr);
trace!("Ignoring dial attempt for {} because it is not a websocket multiaddr",
original_addr);
return Err((self, original_addr));
}
};
debug!(target: "libp2p-websocket", "Dialing {} through inner transport", inner_addr);
debug!("Dialing {} through inner transport", inner_addr);
let ws_addr = client_addr_to_ws(&inner_addr, is_wss);
let inner_dial = match self.transport.dial(inner_addr) {
Ok(d) => d,
Err((transport, old_addr)) => {
warn!(target: "libp2p-websocket", "Failed to dial {} because {} is not supported \
by the underlying transport", original_addr,
old_addr);
warn!("Failed to dial {} because {} is not supported by the underlying transport",
original_addr,
old_addr);
return Err((
WsConfig {
transport: transport,
@ -189,8 +188,7 @@ where
.async_connect_on(connec)
.map_err(|err| IoError::new(IoErrorKind::Other, err))
.map(|(client, _)| {
debug!(target: "libp2p-websocket", "Upgraded outgoing connection to \
websockets");
debug!("Upgraded outgoing connection to websockets");
// Plug our own API on top of the API of the websockets library.
let framed_data = client