policy_client.py 13 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374
  1. """REST client to interact with a policy server.
  2. This client supports both local and remote policy inference modes. Local
  3. inference is faster but causes more compute to be done on the client.
  4. """
  5. import logging
  6. import threading
  7. import time
  8. from typing import Union, Optional
  9. import ray.cloudpickle as pickle
  10. from ray.rllib.env.external_env import ExternalEnv
  11. from ray.rllib.env.external_multi_agent_env import ExternalMultiAgentEnv
  12. from ray.rllib.env.multi_agent_env import MultiAgentEnv
  13. from ray.rllib.policy.sample_batch import MultiAgentBatch
  14. from ray.rllib.utils.annotations import PublicAPI
  15. from ray.rllib.utils.multi_agent import check_multi_agent
  16. from ray.rllib.utils.typing import MultiAgentDict, EnvInfoDict, EnvObsType, \
  17. EnvActionType
  18. logger = logging.getLogger(__name__)
  19. try:
  20. import requests # `requests` is not part of stdlib.
  21. except ImportError:
  22. requests = None
  23. logger.warning(
  24. "Couldn't import `requests` library. Be sure to install it on"
  25. " the client side.")
  26. @PublicAPI
  27. class PolicyClient:
  28. """REST client to interact with a RLlib policy server."""
  29. # Generic commands (for both modes).
  30. ACTION_SPACE = "ACTION_SPACE"
  31. OBSERVATION_SPACE = "OBSERVATION_SPACE"
  32. # Commands for local inference mode.
  33. GET_WORKER_ARGS = "GET_WORKER_ARGS"
  34. GET_WEIGHTS = "GET_WEIGHTS"
  35. REPORT_SAMPLES = "REPORT_SAMPLES"
  36. # Commands for remote inference mode.
  37. START_EPISODE = "START_EPISODE"
  38. GET_ACTION = "GET_ACTION"
  39. LOG_ACTION = "LOG_ACTION"
  40. LOG_RETURNS = "LOG_RETURNS"
  41. END_EPISODE = "END_EPISODE"
  42. @PublicAPI
  43. def __init__(self,
  44. address: str,
  45. inference_mode: str = "local",
  46. update_interval: float = 10.0):
  47. """Create a PolicyClient instance.
  48. Args:
  49. address (str): Server to connect to (e.g., "localhost:9090").
  50. inference_mode (str): Whether to use 'local' or 'remote' policy
  51. inference for computing actions.
  52. update_interval (float or None): If using 'local' inference mode,
  53. the policy is refreshed after this many seconds have passed,
  54. or None for manual control via client.
  55. """
  56. self.address = address
  57. self.env: ExternalEnv = None
  58. if inference_mode == "local":
  59. self.local = True
  60. self._setup_local_rollout_worker(update_interval)
  61. elif inference_mode == "remote":
  62. self.local = False
  63. else:
  64. raise ValueError(
  65. "inference_mode must be either 'local' or 'remote'")
  66. @PublicAPI
  67. def start_episode(self,
  68. episode_id: Optional[str] = None,
  69. training_enabled: bool = True) -> str:
  70. """Record the start of one or more episode(s).
  71. Args:
  72. episode_id (Optional[str]): Unique string id for the episode or
  73. None for it to be auto-assigned.
  74. training_enabled (bool): Whether to use experiences for this
  75. episode to improve the policy.
  76. Returns:
  77. episode_id (str): Unique string id for the episode.
  78. """
  79. if self.local:
  80. self._update_local_policy()
  81. return self.env.start_episode(episode_id, training_enabled)
  82. return self._send({
  83. "episode_id": episode_id,
  84. "command": PolicyClient.START_EPISODE,
  85. "training_enabled": training_enabled,
  86. })["episode_id"]
  87. @PublicAPI
  88. def get_action(self, episode_id: str,
  89. observation: Union[EnvObsType, MultiAgentDict]
  90. ) -> Union[EnvActionType, MultiAgentDict]:
  91. """Record an observation and get the on-policy action.
  92. Args:
  93. episode_id (str): Episode id returned from start_episode().
  94. observation (obj): Current environment observation.
  95. Returns:
  96. action (obj): Action from the env action space.
  97. """
  98. if self.local:
  99. self._update_local_policy()
  100. if isinstance(episode_id, (list, tuple)):
  101. actions = {
  102. eid: self.env.get_action(eid, observation[eid])
  103. for eid in episode_id
  104. }
  105. return actions
  106. else:
  107. return self.env.get_action(episode_id, observation)
  108. else:
  109. return self._send({
  110. "command": PolicyClient.GET_ACTION,
  111. "observation": observation,
  112. "episode_id": episode_id,
  113. })["action"]
  114. @PublicAPI
  115. def log_action(self, episode_id: str,
  116. observation: Union[EnvObsType, MultiAgentDict],
  117. action: Union[EnvActionType, MultiAgentDict]) -> None:
  118. """Record an observation and (off-policy) action taken.
  119. Args:
  120. episode_id (str): Episode id returned from start_episode().
  121. observation (obj): Current environment observation.
  122. action (obj): Action for the observation.
  123. """
  124. if self.local:
  125. self._update_local_policy()
  126. return self.env.log_action(episode_id, observation, action)
  127. self._send({
  128. "command": PolicyClient.LOG_ACTION,
  129. "observation": observation,
  130. "action": action,
  131. "episode_id": episode_id,
  132. })
  133. @PublicAPI
  134. def log_returns(
  135. self,
  136. episode_id: str,
  137. reward: int,
  138. info: Union[EnvInfoDict, MultiAgentDict] = None,
  139. multiagent_done_dict: Optional[MultiAgentDict] = None) -> None:
  140. """Record returns from the environment.
  141. The reward will be attributed to the previous action taken by the
  142. episode. Rewards accumulate until the next action. If no reward is
  143. logged before the next action, a reward of 0.0 is assumed.
  144. Args:
  145. episode_id (str): Episode id returned from start_episode().
  146. reward (float): Reward from the environment.
  147. info (dict): Extra info dict.
  148. multiagent_done_dict (dict): Multi-agent done information.
  149. """
  150. if self.local:
  151. self._update_local_policy()
  152. if multiagent_done_dict is not None:
  153. assert isinstance(reward, dict)
  154. return self.env.log_returns(episode_id, reward, info,
  155. multiagent_done_dict)
  156. return self.env.log_returns(episode_id, reward, info)
  157. self._send({
  158. "command": PolicyClient.LOG_RETURNS,
  159. "reward": reward,
  160. "info": info,
  161. "episode_id": episode_id,
  162. "done": multiagent_done_dict,
  163. })
  164. @PublicAPI
  165. def end_episode(self, episode_id: str,
  166. observation: Union[EnvObsType, MultiAgentDict]) -> None:
  167. """Record the end of an episode.
  168. Args:
  169. episode_id (str): Episode id returned from start_episode().
  170. observation (obj): Current environment observation.
  171. """
  172. if self.local:
  173. self._update_local_policy()
  174. return self.env.end_episode(episode_id, observation)
  175. self._send({
  176. "command": PolicyClient.END_EPISODE,
  177. "observation": observation,
  178. "episode_id": episode_id,
  179. })
  180. @PublicAPI
  181. def update_policy_weights(self) -> None:
  182. """Query the server for new policy weights, if local inference is enabled.
  183. """
  184. self._update_local_policy(force=True)
  185. def _send(self, data):
  186. payload = pickle.dumps(data)
  187. response = requests.post(self.address, data=payload)
  188. if response.status_code != 200:
  189. logger.error("Request failed {}: {}".format(response.text, data))
  190. response.raise_for_status()
  191. parsed = pickle.loads(response.content)
  192. return parsed
  193. def _setup_local_rollout_worker(self, update_interval):
  194. self.update_interval = update_interval
  195. self.last_updated = 0
  196. logger.info("Querying server for rollout worker settings.")
  197. kwargs = self._send({
  198. "command": PolicyClient.GET_WORKER_ARGS,
  199. })["worker_args"]
  200. (self.rollout_worker,
  201. self.inference_thread) = _create_embedded_rollout_worker(
  202. kwargs, self._send)
  203. self.env = self.rollout_worker.env
  204. def _update_local_policy(self, force=False):
  205. assert self.inference_thread.is_alive()
  206. if (self.update_interval and time.time() - self.last_updated >
  207. self.update_interval) or force:
  208. logger.info("Querying server for new policy weights.")
  209. resp = self._send({
  210. "command": PolicyClient.GET_WEIGHTS,
  211. })
  212. weights = resp["weights"]
  213. global_vars = resp["global_vars"]
  214. logger.info(
  215. "Updating rollout worker weights and global vars {}.".format(
  216. global_vars))
  217. self.rollout_worker.set_weights(weights, global_vars)
  218. self.last_updated = time.time()
  219. class _LocalInferenceThread(threading.Thread):
  220. """Thread that handles experience generation (worker.sample() loop)."""
  221. def __init__(self, rollout_worker, send_fn):
  222. super().__init__()
  223. self.daemon = True
  224. self.rollout_worker = rollout_worker
  225. self.send_fn = send_fn
  226. def run(self):
  227. try:
  228. while True:
  229. logger.info("Generating new batch of experiences.")
  230. samples = self.rollout_worker.sample()
  231. metrics = self.rollout_worker.get_metrics()
  232. if isinstance(samples, MultiAgentBatch):
  233. logger.info(
  234. "Sending batch of {} env steps ({} agent steps) to "
  235. "server.".format(samples.env_steps(),
  236. samples.agent_steps()))
  237. else:
  238. logger.info(
  239. "Sending batch of {} steps back to server.".format(
  240. samples.count))
  241. self.send_fn({
  242. "command": PolicyClient.REPORT_SAMPLES,
  243. "samples": samples,
  244. "metrics": metrics,
  245. })
  246. except Exception as e:
  247. logger.info("Error: inference worker thread died!", e)
  248. def _auto_wrap_external(real_env_creator):
  249. """Wrap an environment in the ExternalEnv interface if needed.
  250. Args:
  251. real_env_creator (fn): Create an env given the env_config.
  252. """
  253. def wrapped_creator(env_config):
  254. real_env = real_env_creator(env_config)
  255. if not isinstance(real_env, (ExternalEnv, ExternalMultiAgentEnv)):
  256. logger.info(
  257. "The env you specified is not a supported (sub-)type of "
  258. "ExternalEnv. Attempting to convert it automatically to "
  259. "ExternalEnv.")
  260. if isinstance(real_env, MultiAgentEnv):
  261. external_cls = ExternalMultiAgentEnv
  262. else:
  263. external_cls = ExternalEnv
  264. class ExternalEnvWrapper(external_cls):
  265. def __init__(self, real_env):
  266. super().__init__(
  267. observation_space=real_env.observation_space,
  268. action_space=real_env.action_space)
  269. def run(self):
  270. # Since we are calling methods on this class in the
  271. # client, run doesn't need to do anything.
  272. time.sleep(999999)
  273. return ExternalEnvWrapper(real_env)
  274. return real_env
  275. return wrapped_creator
  276. def _create_embedded_rollout_worker(kwargs, send_fn):
  277. """Create a local rollout worker and a thread that samples from it.
  278. Args:
  279. kwargs (dict): args for the RolloutWorker constructor.
  280. send_fn (fn): function to send a JSON request to the server.
  281. """
  282. # Since the server acts as an input datasource, we have to reset the
  283. # input config to the default, which runs env rollouts.
  284. kwargs = kwargs.copy()
  285. del kwargs["input_creator"]
  286. # Since the server also acts as an output writer, we might have to reset
  287. # the output config to the default, i.e. "output": None, otherwise a
  288. # local rollout worker might write to an unknown output directory
  289. del kwargs["output_creator"]
  290. # If server has no env (which is the expected case):
  291. # Generate a dummy ExternalEnv here using RandomEnv and the
  292. # given observation/action spaces.
  293. if kwargs["policy_config"].get("env") is None:
  294. from ray.rllib.examples.env.random_env import RandomEnv, \
  295. RandomMultiAgentEnv
  296. config = {
  297. "action_space": kwargs["policy_config"]["action_space"],
  298. "observation_space": kwargs["policy_config"]["observation_space"],
  299. }
  300. _, is_ma = check_multi_agent(kwargs["policy_config"])
  301. kwargs["env_creator"] = _auto_wrap_external(
  302. lambda _: (RandomMultiAgentEnv if is_ma else RandomEnv)(config))
  303. kwargs["policy_config"]["env"] = True
  304. # Otherwise, use the env specified by the server args.
  305. else:
  306. real_env_creator = kwargs["env_creator"]
  307. kwargs["env_creator"] = _auto_wrap_external(real_env_creator)
  308. logger.info("Creating rollout worker with kwargs={}".format(kwargs))
  309. from ray.rllib.evaluation.rollout_worker import RolloutWorker
  310. rollout_worker = RolloutWorker(**kwargs)
  311. inference_thread = _LocalInferenceThread(rollout_worker, send_fn)
  312. inference_thread.start()
  313. return rollout_worker, inference_thread